You are viewing a plain text version of this content. The canonical link for it is here.
Posted to issues@ozone.apache.org by GitBox <gi...@apache.org> on 2020/03/11 08:37:43 UTC

[GitHub] [hadoop-ozone] timmylicheng opened a new pull request #663: HDDS-3156 update allocateContainer to remove additional createPipeline step.

timmylicheng opened a new pull request #663: HDDS-3156 update allocateContainer to remove additional createPipeline step.
URL: https://github.com/apache/hadoop-ozone/pull/663
 
 
   ## What changes were proposed in this pull request?
   Upate allocateContainer not to try creating pipeline every time.
   (Please fill in changes proposed in this fix)
   
   ## What is the link to the Apache JIRA
   https://issues.apache.org/jira/browse/HDDS-3156
   (Please create an issue in ASF JIRA before opening a pull request,
   and you need to set the title of the pull request which starts with
   the corresponding JIRA issue number. (e.g. HDDS-XXXX. Fix a typo in YYY.)
   
   Please replace this section with the link to the Apache JIRA)
   
   ## How was this patch tested?
   Not needed.
   (Please explain how this patch was tested. Ex: unit tests, manual tests)
   (If this patch involves UI changes, please attach a screen-shot; otherwise, remove this)
   

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

---------------------------------------------------------------------
To unsubscribe, e-mail: ozone-issues-unsubscribe@hadoop.apache.org
For additional commands, e-mail: ozone-issues-help@hadoop.apache.org


[GitHub] [hadoop-ozone] timmylicheng commented on a change in pull request #663: HDDS-3156 update allocateContainer to remove additional createPipeline step.

Posted by GitBox <gi...@apache.org>.
timmylicheng commented on a change in pull request #663: HDDS-3156 update allocateContainer to remove additional createPipeline step.
URL: https://github.com/apache/hadoop-ozone/pull/663#discussion_r391472796
 
 

 ##########
 File path: hadoop-hdds/server-scm/src/main/java/org/apache/hadoop/hdds/scm/container/ContainerStateManager.java
 ##########
 @@ -247,23 +247,29 @@ ContainerInfo allocateContainer(final PipelineManager pipelineManager,
       final HddsProtos.ReplicationType type,
       final HddsProtos.ReplicationFactor replicationFactor, final String owner)
       throws IOException {
-
+    final List<Pipeline> pipelines = pipelineManager
+        .getPipelines(type, replicationFactor, Pipeline.PipelineState.OPEN);
     Pipeline pipeline;
-    try {
-      // TODO: #CLUTIL remove creation logic when all replication types and
-      // factors are handled by pipeline creator job.
-      pipeline = pipelineManager.createPipeline(type, replicationFactor);
-      pipelineManager.waitPipelineReady(pipeline.getId(), 0);
-    } catch (IOException e) {
-      final List<Pipeline> pipelines = pipelineManager
-          .getPipelines(type, replicationFactor, Pipeline.PipelineState.OPEN);
-      if (pipelines.isEmpty()) {
-        throw new IOException("Could not allocate container. Cannot get any" +
-            " matching pipeline for Type:" + type +
-            ", Factor:" + replicationFactor + ", State:PipelineState.OPEN");
-      }
+
+    if (!pipelines.isEmpty() && replicationFactor == ReplicationFactor.THREE
+        && type == ReplicationType.RATIS) {
+      // Let background create Ratis THREE pipelines.
 
 Review comment:
   Yea, that's a good point. I update it.

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

---------------------------------------------------------------------
To unsubscribe, e-mail: ozone-issues-unsubscribe@hadoop.apache.org
For additional commands, e-mail: ozone-issues-help@hadoop.apache.org


[GitHub] [hadoop-ozone] nandakumar131 merged pull request #663: HDDS-3156 update allocateContainer to remove additional createPipeline step.

Posted by GitBox <gi...@apache.org>.
nandakumar131 merged pull request #663: HDDS-3156 update allocateContainer to remove additional createPipeline step.
URL: https://github.com/apache/hadoop-ozone/pull/663
 
 
   

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

---------------------------------------------------------------------
To unsubscribe, e-mail: ozone-issues-unsubscribe@hadoop.apache.org
For additional commands, e-mail: ozone-issues-help@hadoop.apache.org


[GitHub] [hadoop-ozone] nandakumar131 commented on issue #663: HDDS-3156 update allocateContainer to remove additional createPipeline step.

Posted by GitBox <gi...@apache.org>.
nandakumar131 commented on issue #663: HDDS-3156 update allocateContainer to remove additional createPipeline step.
URL: https://github.com/apache/hadoop-ozone/pull/663#issuecomment-600727022
 
 
   @timmylicheng  thanks for the clarification.
   
   As we create the pipeline via heartbeat now it's not good to have it in the critical path; unless there are no pipelines available, we should try to avoid the creation of pipelines in allocate container calls and leave the job to background pipeline creator.
   
   I can see that changing all the related tests seems hectic and we should also have a discussion before making one node pipeline creation default in background pipeline creator.
   
   I'm good with the current changes for now. We can revisit this later.

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

---------------------------------------------------------------------
To unsubscribe, e-mail: ozone-issues-unsubscribe@hadoop.apache.org
For additional commands, e-mail: ozone-issues-help@hadoop.apache.org


[GitHub] [hadoop-ozone] timmylicheng commented on issue #663: HDDS-3156 update allocateContainer to remove additional createPipeline step.

Posted by GitBox <gi...@apache.org>.
timmylicheng commented on issue #663: HDDS-3156 update allocateContainer to remove additional createPipeline step.
URL: https://github.com/apache/hadoop-ozone/pull/663#issuecomment-597735438
 
 
   This is going to let Ratis Three pipeline creation be handled only in BackgroundPipelineCreator. For all types or factor of pipelines, it still stay the same behavior.

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

---------------------------------------------------------------------
To unsubscribe, e-mail: ozone-issues-unsubscribe@hadoop.apache.org
For additional commands, e-mail: ozone-issues-help@hadoop.apache.org


[GitHub] [hadoop-ozone] nandakumar131 commented on issue #663: HDDS-3156 update allocateContainer to remove additional createPipeline step.

Posted by GitBox <gi...@apache.org>.
nandakumar131 commented on issue #663: HDDS-3156 update allocateContainer to remove additional createPipeline step.
URL: https://github.com/apache/hadoop-ozone/pull/663#issuecomment-606771274
 
 
   Thanks @timmylicheng for the patch and thanks @bharatviswa504  for the review. I will merge this shortly.

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

---------------------------------------------------------------------
To unsubscribe, e-mail: ozone-issues-unsubscribe@hadoop.apache.org
For additional commands, e-mail: ozone-issues-help@hadoop.apache.org


[GitHub] [hadoop-ozone] nandakumar131 commented on issue #663: HDDS-3156 update allocateContainer to remove additional createPipeline step.

Posted by GitBox <gi...@apache.org>.
nandakumar131 commented on issue #663: HDDS-3156 update allocateContainer to remove additional createPipeline step.
URL: https://github.com/apache/hadoop-ozone/pull/663#issuecomment-600269488
 
 
   
   Thanks @timmylicheng for working on this.
   
   We don't need any complex conditions here. The below logic should be enough to handle
   
   ```
   try {
     final List<Pipeline> pipelines = pipelineManager.getPipelines(type, replicationFactor, Pipeline.PipelineState.OPEN);
     Pipeline pipeline;
     if (pipelines.isEmpty()) {
       pipeline = pipelineManager.createPipeline(type, replicationFactor);
       pipelineManager.waitPipelineReady(pipeline.getId(), 0);
     } else {
       pipeline = pipelines.get((int) containerCount.get() % pipelines.size());
     }
   } catch (IOException e) {
     throw new IOException("Could not allocate container. Cannot get any" +
                 " matching pipeline for Type:" + type +
                 ", Factor:" + replicationFactor + ", State:PipelineState.OPEN");
   }
   ```

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

---------------------------------------------------------------------
To unsubscribe, e-mail: ozone-issues-unsubscribe@hadoop.apache.org
For additional commands, e-mail: ozone-issues-help@hadoop.apache.org


[GitHub] [hadoop-ozone] bharatviswa504 commented on a change in pull request #663: HDDS-3156 update allocateContainer to remove additional createPipeline step.

Posted by GitBox <gi...@apache.org>.
bharatviswa504 commented on a change in pull request #663: HDDS-3156 update allocateContainer to remove additional createPipeline step.
URL: https://github.com/apache/hadoop-ozone/pull/663#discussion_r391349507
 
 

 ##########
 File path: hadoop-hdds/server-scm/src/main/java/org/apache/hadoop/hdds/scm/container/ContainerStateManager.java
 ##########
 @@ -247,23 +247,29 @@ ContainerInfo allocateContainer(final PipelineManager pipelineManager,
       final HddsProtos.ReplicationType type,
       final HddsProtos.ReplicationFactor replicationFactor, final String owner)
       throws IOException {
-
+    final List<Pipeline> pipelines = pipelineManager
+        .getPipelines(type, replicationFactor, Pipeline.PipelineState.OPEN);
     Pipeline pipeline;
-    try {
-      // TODO: #CLUTIL remove creation logic when all replication types and
-      // factors are handled by pipeline creator job.
-      pipeline = pipelineManager.createPipeline(type, replicationFactor);
-      pipelineManager.waitPipelineReady(pipeline.getId(), 0);
-    } catch (IOException e) {
-      final List<Pipeline> pipelines = pipelineManager
-          .getPipelines(type, replicationFactor, Pipeline.PipelineState.OPEN);
-      if (pipelines.isEmpty()) {
-        throw new IOException("Could not allocate container. Cannot get any" +
-            " matching pipeline for Type:" + type +
-            ", Factor:" + replicationFactor + ", State:PipelineState.OPEN");
-      }
+
+    if (!pipelines.isEmpty() && replicationFactor == ReplicationFactor.THREE
+        && type == ReplicationType.RATIS) {
+      // Let background create Ratis THREE pipelines.
 
 Review comment:
   If ozone.scm.pipeline.creation.auto.factor.one is not disabled even ratis with factor one pipeline creation is taken care by BackGroundPipelineCreator.

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

---------------------------------------------------------------------
To unsubscribe, e-mail: ozone-issues-unsubscribe@hadoop.apache.org
For additional commands, e-mail: ozone-issues-help@hadoop.apache.org


[GitHub] [hadoop-ozone] timmylicheng commented on issue #663: HDDS-3156 update allocateContainer to remove additional createPipeline step.

Posted by GitBox <gi...@apache.org>.
timmylicheng commented on issue #663: HDDS-3156 update allocateContainer to remove additional createPipeline step.
URL: https://github.com/apache/hadoop-ozone/pull/663#issuecomment-600406625
 
 
   > Thanks @timmylicheng for working on this.
   > 
   > We don't need any complex conditions here. The below logic should be enough to handle
   > 
   > ```
   > try {
   >   final List<Pipeline> pipelines = pipelineManager.getPipelines(type, replicationFactor, Pipeline.PipelineState.OPEN);
   >   Pipeline pipeline;
   >   if (pipelines.isEmpty()) {
   >     pipeline = pipelineManager.createPipeline(type, replicationFactor);
   >     pipelineManager.waitPipelineReady(pipeline.getId(), 0);
   >   } else {
   >     pipeline = pipelines.get((int) containerCount.get() % pipelines.size());
   >   }
   > } catch (IOException e) {
   >   throw new IOException("Could not allocate container. Cannot get any" +
   >               " matching pipeline for Type:" + type +
   >               ", Factor:" + replicationFactor + ", State:PipelineState.OPEN");
   > }
   > ```
   
   @nandakumar131 I actually tried this and some tests fail. Basically some uts are expecting to have a new Ratis ONE pipeline when a new container is created. In your posted code, it will use an existing pipeline by default. 
   My idea is to keep Ratis One behavior as it is: try to create pipeline every time a container is created. After pipeline is used up for creation, it will use the existing ones. Your idea seems to be using existing pipelines for both Ratis one and Ratis three. But like Bharat posts, Ratis one doesn't have to be auto created by background thread. We do have conditions when Ratis one is not auto created by a config. Also for StandAlone pipelines, background thread is also not working for them.
   I was thinking of changing the least UTs. Would you suggest to modify UTs on the other hand? That could be a lot tho.

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

---------------------------------------------------------------------
To unsubscribe, e-mail: ozone-issues-unsubscribe@hadoop.apache.org
For additional commands, e-mail: ozone-issues-help@hadoop.apache.org


[GitHub] [hadoop-ozone] timmylicheng commented on issue #663: HDDS-3156 update allocateContainer to remove additional createPipeline step.

Posted by GitBox <gi...@apache.org>.
timmylicheng commented on issue #663: HDDS-3156 update allocateContainer to remove additional createPipeline step.
URL: https://github.com/apache/hadoop-ozone/pull/663#issuecomment-602405476
 
 
   > @timmylicheng thanks for the clarification.
   > 
   > As we create the pipeline via heartbeat now it's not good to have it in the critical path; unless there are no pipelines available, we should try to avoid the creation of pipelines in allocate container calls and leave the job to background pipeline creator.
   > 
   > I can see that changing all the related tests seems hectic and we should also have a discussion before making one node pipeline creation default in background pipeline creator.
   > 
   > I'm good with the current changes for now. We can revisit this later.
   
   @nandakumar131 Sounds fair to me. Let's revisit the Ratis One pipeline later since it's not blocking anyone's use cases in real cluster. 

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

---------------------------------------------------------------------
To unsubscribe, e-mail: ozone-issues-unsubscribe@hadoop.apache.org
For additional commands, e-mail: ozone-issues-help@hadoop.apache.org


[GitHub] [hadoop-ozone] bharatviswa504 commented on a change in pull request #663: HDDS-3156 update allocateContainer to remove additional createPipeline step.

Posted by GitBox <gi...@apache.org>.
bharatviswa504 commented on a change in pull request #663: HDDS-3156 update allocateContainer to remove additional createPipeline step.
URL: https://github.com/apache/hadoop-ozone/pull/663#discussion_r391349687
 
 

 ##########
 File path: hadoop-hdds/server-scm/src/main/java/org/apache/hadoop/hdds/scm/container/ContainerStateManager.java
 ##########
 @@ -247,23 +247,29 @@ ContainerInfo allocateContainer(final PipelineManager pipelineManager,
       final HddsProtos.ReplicationType type,
       final HddsProtos.ReplicationFactor replicationFactor, final String owner)
       throws IOException {
-
+    final List<Pipeline> pipelines = pipelineManager
+        .getPipelines(type, replicationFactor, Pipeline.PipelineState.OPEN);
     Pipeline pipeline;
-    try {
-      // TODO: #CLUTIL remove creation logic when all replication types and
-      // factors are handled by pipeline creator job.
-      pipeline = pipelineManager.createPipeline(type, replicationFactor);
-      pipelineManager.waitPipelineReady(pipeline.getId(), 0);
-    } catch (IOException e) {
-      final List<Pipeline> pipelines = pipelineManager
-          .getPipelines(type, replicationFactor, Pipeline.PipelineState.OPEN);
-      if (pipelines.isEmpty()) {
-        throw new IOException("Could not allocate container. Cannot get any" +
-            " matching pipeline for Type:" + type +
-            ", Factor:" + replicationFactor + ", State:PipelineState.OPEN");
-      }
+
+    if (!pipelines.isEmpty() && replicationFactor == ReplicationFactor.THREE
+        && type == ReplicationType.RATIS) {
+      // Let background create Ratis THREE pipelines.
 
 Review comment:
   So, shall we need to check that config, if enabled and let factor 1 also be skipped here?

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

---------------------------------------------------------------------
To unsubscribe, e-mail: ozone-issues-unsubscribe@hadoop.apache.org
For additional commands, e-mail: ozone-issues-help@hadoop.apache.org