Browse Source

HDDS-1475 : Fix OzoneContainer start method. (#788)

avijayanhwx 6 years ago
parent
commit
eb9c8900bc

+ 9 - 2
hadoop-hdds/container-service/src/main/java/org/apache/hadoop/ozone/container/common/transport/server/XceiverServerGrpc.java

@@ -69,6 +69,7 @@ public final class XceiverServerGrpc extends XceiverServer {
   private UUID id;
   private Server server;
   private final ContainerDispatcher storageContainer;
+  private boolean isStarted;
 
   /**
    * Constructs a Grpc server class.
@@ -161,12 +162,18 @@ public final class XceiverServerGrpc extends XceiverServer {
 
   @Override
   public void start() throws IOException {
-    server.start();
+    if (!isStarted) {
+      server.start();
+      isStarted = true;
+    }
   }
 
   @Override
   public void stop() {
-    server.shutdown();
+    if (isStarted) {
+      server.shutdown();
+      isStarted = false;
+    }
   }
 
   @Override

+ 19 - 12
hadoop-hdds/container-service/src/main/java/org/apache/hadoop/ozone/container/common/transport/server/ratis/XceiverServerRatis.java

@@ -111,6 +111,7 @@ public final class XceiverServerRatis extends XceiverServer {
   private final ReplicationLevel replicationLevel;
   private long nodeFailureTimeoutMs;
   private final long cacheEntryExpiryInteval;
+  private boolean isStarted = false;
 
   private XceiverServerRatis(DatanodeDetails dd, int port,
       ContainerDispatcher dispatcher, Configuration conf, StateContext
@@ -413,22 +414,28 @@ public final class XceiverServerRatis extends XceiverServer {
 
   @Override
   public void start() throws IOException {
-    LOG.info("Starting {} {} at port {}", getClass().getSimpleName(),
-        server.getId(), getIPCPort());
-    chunkExecutor.prestartAllCoreThreads();
-    server.start();
+    if (!isStarted) {
+      LOG.info("Starting {} {} at port {}", getClass().getSimpleName(),
+          server.getId(), getIPCPort());
+      chunkExecutor.prestartAllCoreThreads();
+      server.start();
+      isStarted = true;
+    }
   }
 
   @Override
   public void stop() {
-    try {
-      // shutdown server before the executors as while shutting down,
-      // some of the tasks would be executed using the executors.
-      server.close();
-      chunkExecutor.shutdown();
-      executors.forEach(ExecutorService::shutdown);
-    } catch (IOException e) {
-      throw new RuntimeException(e);
+    if (isStarted) {
+      try {
+        // shutdown server before the executors as while shutting down,
+        // some of the tasks would be executed using the executors.
+        server.close();
+        chunkExecutor.shutdown();
+        executors.forEach(ExecutorService::shutdown);
+        isStarted = false;
+      } catch (IOException e) {
+        throw new RuntimeException(e);
+      }
     }
   }
 

+ 3 - 1
hadoop-hdds/container-service/src/main/java/org/apache/hadoop/ozone/container/ozoneimpl/OzoneContainer.java

@@ -160,7 +160,9 @@ public class OzoneContainer {
       LOG.info("Background container scrubber has been disabled by {}",
               HddsConfigKeys.HDDS_CONTAINERSCRUB_ENABLED);
     } else {
-      this.scrubber = new ContainerScrubber(containerSet, config);
+      if (this.scrubber == null) {
+        this.scrubber = new ContainerScrubber(containerSet, config);
+      }
       scrubber.up();
     }
   }

+ 53 - 0
hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/container/ozoneimpl/TestOzoneContainer.java

@@ -98,6 +98,59 @@ public class TestOzoneContainer {
     }
   }
 
+  @Test
+  public void testOzoneContainerStart() throws Exception {
+    OzoneConfiguration conf = newOzoneConfiguration();
+    MiniOzoneCluster cluster = null;
+    OzoneContainer container = null;
+
+    try {
+      cluster = MiniOzoneCluster.newBuilder(conf).build();
+      cluster.waitForClusterToBeReady();
+
+      Pipeline pipeline = ContainerTestHelper.createSingleNodePipeline();
+      conf.set(HDDS_DATANODE_DIR_KEY, tempFolder.getRoot().getPath());
+      conf.setInt(OzoneConfigKeys.DFS_CONTAINER_IPC_PORT,
+          pipeline.getFirstNode()
+              .getPort(DatanodeDetails.Port.Name.STANDALONE).getValue());
+      conf.setBoolean(
+          OzoneConfigKeys.DFS_CONTAINER_IPC_RANDOM_PORT, false);
+
+
+      DatanodeDetails datanodeDetails = TestUtils.randomDatanodeDetails();
+      StateContext context = Mockito.mock(StateContext.class);
+      DatanodeStateMachine dsm = Mockito.mock(DatanodeStateMachine.class);
+      Mockito.when(dsm.getDatanodeDetails()).thenReturn(datanodeDetails);
+      Mockito.when(context.getParent()).thenReturn(dsm);
+      container = new OzoneContainer(datanodeDetails, conf,
+          context, null);
+
+      String scmId = UUID.randomUUID().toString();
+      container.start(scmId);
+      try {
+        container.start(scmId);
+      } catch (Exception e) {
+        Assert.fail();
+      }
+
+      container.stop();
+      try {
+        container.stop();
+      } catch (Exception e) {
+        Assert.fail();
+      }
+
+    } finally {
+      if (container != null) {
+        container.stop();
+      }
+      if (cluster != null) {
+        cluster.shutdown();
+      }
+    }
+  }
+
+
   static OzoneConfiguration newOzoneConfiguration() {
     final OzoneConfiguration conf = new OzoneConfiguration();
     return conf;