Explorar el Código

HDDS-1053. Generate RaftGroupId from OMServiceID. Contributed by Aravindan Vijayan.

Arpit Agarwal hace 6 años
padre
commit
676a9cb888

+ 1 - 1
hadoop-hdds/common/src/main/java/org/apache/hadoop/ozone/OzoneConsts.java

@@ -273,5 +273,5 @@ public final class OzoneConsts {
       Metadata.Key.of(OZONE_USER, ASCII_STRING_MARSHALLER);
 
   // Default OMServiceID for OM Ratis servers to use as RaftGroupId
-  public static final String OM_SERVICE_ID_DEFAULT = "om-service-value";
+  public static final String OM_SERVICE_ID_DEFAULT = "omServiceIdDefault";
 }

+ 6 - 2
hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/ratis/OzoneManagerRatisServer.java

@@ -26,6 +26,7 @@ import java.net.InetSocketAddress;
 import java.util.ArrayList;
 import java.util.Collections;
 import java.util.List;
+import java.util.UUID;
 import java.util.concurrent.TimeUnit;
 
 import org.apache.hadoop.conf.Configuration;
@@ -48,7 +49,6 @@ import org.apache.ratis.rpc.SupportedRpcType;
 import org.apache.ratis.server.RaftServer;
 import org.apache.ratis.server.RaftServerConfigKeys;
 import org.apache.ratis.statemachine.impl.BaseStateMachine;
-import org.apache.ratis.thirdparty.com.google.protobuf.ByteString;
 import org.apache.ratis.util.LifeCycle;
 import org.apache.ratis.util.SizeInBytes;
 import org.apache.ratis.util.TimeDuration;
@@ -91,7 +91,7 @@ public final class OzoneManagerRatisServer {
 
     this.raftPeerId = localRaftPeerId;
     this.raftGroupId = RaftGroupId.valueOf(
-        ByteString.copyFromUtf8(raftGroupIdStr));
+        getRaftGroupIdFromOmServiceId(raftGroupIdStr));
     this.raftGroup = RaftGroup.valueOf(raftGroupId, raftPeers);
 
     StringBuilder raftPeersStr = new StringBuilder();
@@ -355,4 +355,8 @@ public final class OzoneManagerRatisServer {
     }
     return storageDir;
   }
+
+  private UUID getRaftGroupIdFromOmServiceId(String omServiceId) {
+    return UUID.nameUUIDFromBytes(omServiceId.getBytes());
+  }
 }

+ 49 - 0
hadoop-ozone/ozone-manager/src/test/java/org/apache/hadoop/ozone/om/ratis/TestOzoneManagerRatisServer.java

@@ -38,6 +38,7 @@ import org.apache.hadoop.ozone.protocol.proto.OzoneManagerProtocolProtos
     .OMResponse;
 import org.apache.hadoop.ozone.protocol.proto.OzoneManagerProtocolProtos;
 import org.apache.hadoop.test.GenericTestUtils;
+import org.apache.ratis.protocol.RaftGroupId;
 import org.apache.ratis.util.LifeCycle;
 import org.junit.After;
 import org.junit.Assert;
@@ -152,4 +153,52 @@ public class TestOzoneManagerRatisServer {
       logCapturer.clearOutput();
     }
   }
+
+  @Test
+  public void verifyRaftGroupIdGenerationWithDefaultOmServiceId() throws
+      Exception {
+    UUID uuid = UUID.nameUUIDFromBytes(OzoneConsts.OM_SERVICE_ID_DEFAULT
+        .getBytes());
+    RaftGroupId raftGroupId = omRatisServer.getRaftGroup().getGroupId();
+    Assert.assertEquals(uuid, raftGroupId.getUuid());
+    Assert.assertEquals(raftGroupId.toByteString().size(), 16);
+  }
+
+  @Test
+  public void verifyRaftGroupIdGenerationWithCustomOmServiceId() throws
+      Exception {
+    String customOmServiceId = "omSIdCustom123";
+    OzoneConfiguration newConf = new OzoneConfiguration();
+    String newOmId = UUID.randomUUID().toString();
+    String path = GenericTestUtils.getTempPath(newOmId);
+    Path metaDirPath = Paths.get(path, "om-meta");
+    newConf.set(HddsConfigKeys.OZONE_METADATA_DIRS, metaDirPath.toString());
+    newConf.setTimeDuration(
+        OMConfigKeys.OZONE_OM_LEADER_ELECTION_MINIMUM_TIMEOUT_DURATION_KEY,
+        LEADER_ELECTION_TIMEOUT, TimeUnit.MILLISECONDS);
+    int ratisPort = 9873;
+    InetSocketAddress rpcAddress = new InetSocketAddress(
+        InetAddress.getLocalHost(), 0);
+    OMNodeDetails omNodeDetails = new OMNodeDetails.Builder()
+        .setRpcAddress(rpcAddress)
+        .setRatisPort(ratisPort)
+        .setOMNodeId(newOmId)
+        .setOMServiceId(customOmServiceId)
+        .build();
+    // Starts a single node Ratis server
+    OzoneManagerRatisServer newOmRatisServer = OzoneManagerRatisServer
+        .newOMRatisServer(newConf, null,
+            omNodeDetails, Collections.emptyList());
+    newOmRatisServer.start();
+    OzoneManagerRatisClient newOmRatisClient = OzoneManagerRatisClient
+        .newOzoneManagerRatisClient(
+            newOmId,
+            newOmRatisServer.getRaftGroup(), newConf);
+    newOmRatisClient.connect();
+
+    UUID uuid = UUID.nameUUIDFromBytes(customOmServiceId.getBytes());
+    RaftGroupId raftGroupId = newOmRatisServer.getRaftGroup().getGroupId();
+    Assert.assertEquals(uuid, raftGroupId.getUuid());
+    Assert.assertEquals(raftGroupId.toByteString().size(), 16);
+  }
 }