浏览代码

HDFS-12795. Ozone: SCM: Support for Container LifeCycleState PENDING_CLOSE and LifeCycleEvent FULL_CONTAINER. Contributed by Nandakumar.

Anu Engineer 7 年之前
父节点
当前提交
966853894f

+ 10 - 8
hadoop-hdfs-project/hadoop-hdfs-client/src/main/proto/Ozone.proto

@@ -104,19 +104,21 @@ enum LifeCycleState {
     ALLOCATED = 1;
     ALLOCATED = 1;
     CREATING = 2; // Used for container allocated/created by different client.
     CREATING = 2; // Used for container allocated/created by different client.
     OPEN =3; // Mostly an update to SCM via HB or client call.
     OPEN =3; // Mostly an update to SCM via HB or client call.
-    CLOSED = 4; // !!State after this has not been used yet.
-    DELETING = 5;
-    DELETED = 6; // object is deleted.
+    PENDING_CLOSE = 4;
+    CLOSED = 5; // !!State after this has not been used yet.
+    DELETING = 6;
+    DELETED = 7; // object is deleted.
 }
 }
 
 
 enum LifeCycleEvent {
 enum LifeCycleEvent {
     BEGIN_CREATE = 1; // A request to client to create this object
     BEGIN_CREATE = 1; // A request to client to create this object
     COMPLETE_CREATE = 2;
     COMPLETE_CREATE = 2;
-    CLOSE = 3; // !!Event after this has not been used yet.
-    UPDATE = 4;
-    TIMEOUT = 5; // creation has timed out from SCM's View.
-    DELETE = 6;
-    CLEANUP = 7;
+    FULL_CONTAINER = 3;
+    CLOSE = 4; // !!Event after this has not been used yet.
+    UPDATE = 5;
+    TIMEOUT = 6; // creation has timed out from SCM's View.
+    DELETE = 7;
+    CLEANUP = 8;
 }
 }
 
 
 message SCMContainerInfo {
 message SCMContainerInfo {

+ 29 - 11
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/ozone/scm/container/ContainerMapping.java

@@ -56,6 +56,7 @@ import java.util.concurrent.locks.Lock;
 import java.util.concurrent.locks.ReentrantLock;
 import java.util.concurrent.locks.ReentrantLock;
 
 
 import static org.apache.hadoop.ozone.OzoneConsts.CONTAINER_DB;
 import static org.apache.hadoop.ozone.OzoneConsts.CONTAINER_DB;
+import static org.apache.hadoop.ozone.scm.exceptions.SCMException.ResultCodes.FAILED_TO_CHANGE_CONTAINER_STATE;
 
 
 /**
 /**
  * Mapping class contains the mapping from a name to a pipeline mapping. This
  * Mapping class contains the mapping from a name to a pipeline mapping. This
@@ -296,8 +297,8 @@ public class ContainerMapping implements Mapping {
               .parseFrom(containerBytes));
               .parseFrom(containerBytes));
 
 
       Preconditions.checkNotNull(containerInfo);
       Preconditions.checkNotNull(containerInfo);
-
-      if (event == OzoneProtos.LifeCycleEvent.BEGIN_CREATE) {
+      switch (event) {
+      case BEGIN_CREATE:
         // Acquire lease on container
         // Acquire lease on container
         Lease<ContainerInfo> containerLease =
         Lease<ContainerInfo> containerLease =
             containerLeaseManager.acquire(containerInfo);
             containerLeaseManager.acquire(containerInfo);
@@ -307,10 +308,30 @@ public class ContainerMapping implements Mapping {
               OzoneProtos.LifeCycleEvent.TIMEOUT);
               OzoneProtos.LifeCycleEvent.TIMEOUT);
           return null;
           return null;
         });
         });
-      } else if (event == OzoneProtos.LifeCycleEvent.COMPLETE_CREATE) {
+        break;
+      case COMPLETE_CREATE:
         // Release the lease on container
         // Release the lease on container
         containerLeaseManager.release(containerInfo);
         containerLeaseManager.release(containerInfo);
+        break;
+      case TIMEOUT:
+        break;
+      case CLEANUP:
+        break;
+      case FULL_CONTAINER:
+        break;
+      case CLOSE:
+        break;
+      case UPDATE:
+        break;
+      case DELETE:
+        break;
+      default:
+        throw new SCMException("Unsupported container LifeCycleEvent.",
+            FAILED_TO_CHANGE_CONTAINER_STATE);
       }
       }
+      // If the below updateContainerState call fails, we should revert the
+      // changes made in switch case.
+      // Like releasing the lease in case of BEGIN_CREATE.
       ContainerInfo updatedContainer = containerStateManager
       ContainerInfo updatedContainer = containerStateManager
           .updateContainerState(containerInfo, event);
           .updateContainerState(containerInfo, event);
       containerStore.put(dbKey, updatedContainer.getProtobuf().toByteArray());
       containerStore.put(dbKey, updatedContainer.getProtobuf().toByteArray());
@@ -383,16 +404,13 @@ public class ContainerMapping implements Mapping {
             // Close container implementation can decide on how to maintain
             // Close container implementation can decide on how to maintain
             // list of containers to be closed, this is the place where we
             // list of containers to be closed, this is the place where we
             // have to add the containers to that list.
             // have to add the containers to that list.
-            ContainerInfo updatedContainer =
-                containerStateManager.updateContainerState(
-                    ContainerInfo.fromProtobuf(newContainerInfo),
-                    OzoneProtos.LifeCycleEvent.CLOSE);
-            if (updatedContainer.getState() !=
-                OzoneProtos.LifeCycleState.CLOSED) {
+            OzoneProtos.LifeCycleState state = updateContainerState(
+                ContainerInfo.fromProtobuf(newContainerInfo).getContainerName(),
+                OzoneProtos.LifeCycleEvent.FULL_CONTAINER);
+            if (state != OzoneProtos.LifeCycleState.PENDING_CLOSE) {
               LOG.error("Failed to close container {}, reason : Not able to " +
               LOG.error("Failed to close container {}, reason : Not able to " +
                       "update container state, current container state: {}." +
                       "update container state, current container state: {}." +
-                      "in state {}", containerInfo.getContainerName(),
-                  updatedContainer.getState());
+                      "in state {}", containerInfo.getContainerName(), state);
             }
             }
           }
           }
         } else {
         } else {

+ 8 - 1
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/ozone/scm/container/ContainerStateManager.java

@@ -251,11 +251,14 @@ public class ContainerStateManager implements Closeable {
   //                                  DELETING----------------->[DELETED]
   //                                  DELETING----------------->[DELETED]
   //                                           (CLEANUP)
   //                                           (CLEANUP)
   // SCM Open/Close State Machine
   // SCM Open/Close State Machine
-  // States: OPEN------------------>PENDING_CLOSE---------->[CLOSE]
+  // States: OPEN------------------>PENDING_CLOSE---------->[CLOSED]
   // Events:        (FULL_CONTAINER)               (CLOSE)
   // Events:        (FULL_CONTAINER)               (CLOSE)
   // Delete State Machine
   // Delete State Machine
   // States: OPEN------------------>DELETING------------------>[DELETED]
   // States: OPEN------------------>DELETING------------------>[DELETED]
   // Events:         (DELETE)                  (CLEANUP)
   // Events:         (DELETE)                  (CLEANUP)
+
+  // Should we allow DELETING of OPEN containers? we can always have
+  // OPEN--------->PENDING_CLOSE----->CLOSE---->DELETING---->[DELETED]
   private void initializeStateMachine() {
   private void initializeStateMachine() {
     stateMachine.addTransition(LifeCycleState.ALLOCATED,
     stateMachine.addTransition(LifeCycleState.ALLOCATED,
         LifeCycleState.CREATING,
         LifeCycleState.CREATING,
@@ -266,6 +269,10 @@ public class ContainerStateManager implements Closeable {
         LifeCycleEvent.COMPLETE_CREATE);
         LifeCycleEvent.COMPLETE_CREATE);
 
 
     stateMachine.addTransition(LifeCycleState.OPEN,
     stateMachine.addTransition(LifeCycleState.OPEN,
+        LifeCycleState.PENDING_CLOSE,
+        LifeCycleEvent.FULL_CONTAINER);
+
+    stateMachine.addTransition(LifeCycleState.PENDING_CLOSE,
         LifeCycleState.CLOSED,
         LifeCycleState.CLOSED,
         LifeCycleEvent.CLOSE);
         LifeCycleEvent.CLOSE);
 
 

+ 28 - 0
hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/ozone/scm/container/TestContainerMapping.java

@@ -264,6 +264,34 @@ public class TestContainerMapping {
     ContainerInfo updatedContainer = mapping.getContainer(containerName);
     ContainerInfo updatedContainer = mapping.getContainer(containerName);
     Assert.assertEquals(500000000L, updatedContainer.getNumberOfKeys());
     Assert.assertEquals(500000000L, updatedContainer.getNumberOfKeys());
     Assert.assertEquals(5368705120L, updatedContainer.getUsedBytes());
     Assert.assertEquals(5368705120L, updatedContainer.getUsedBytes());
+    List<ContainerInfo> pendingCloseContainers = mapping.getStateManager()
+        .getMatchingContainers(
+            OzoneProtos.Owner.OZONE,
+            xceiverClientManager.getType(),
+            xceiverClientManager.getFactor(),
+            OzoneProtos.LifeCycleState.PENDING_CLOSE);
+    Assert.assertTrue(pendingCloseContainers.stream().map(
+        container -> container.getContainerName()).collect(
+        Collectors.toList()).contains(containerName));
+  }
+
+  @Test
+  public void testCloseContainer() throws IOException {
+    String containerName = UUID.randomUUID().toString();
+    createContainer(containerName);
+    mapping.updateContainerState(containerName,
+        OzoneProtos.LifeCycleEvent.FULL_CONTAINER);
+    List<ContainerInfo> pendingCloseContainers = mapping.getStateManager()
+        .getMatchingContainers(
+            OzoneProtos.Owner.OZONE,
+            xceiverClientManager.getType(),
+            xceiverClientManager.getFactor(),
+            OzoneProtos.LifeCycleState.PENDING_CLOSE);
+    Assert.assertTrue(pendingCloseContainers.stream().map(
+        container -> container.getContainerName()).collect(
+        Collectors.toList()).contains(containerName));
+    mapping.updateContainerState(containerName,
+        OzoneProtos.LifeCycleEvent.CLOSE);
     List<ContainerInfo> closeContainers = mapping.getStateManager()
     List<ContainerInfo> closeContainers = mapping.getStateManager()
         .getMatchingContainers(
         .getMatchingContainers(
             OzoneProtos.Owner.OZONE,
             OzoneProtos.Owner.OZONE,