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 2022/02/17 17:32:30 UTC

[GitHub] [ozone] sodonnel opened a new pull request #3110: HDDS-6334. Remove ContainerID to Proto to ContainerID conversion in ContainerStateManagerImpl

sodonnel opened a new pull request #3110:
URL: https://github.com/apache/ozone/pull/3110


   ## What changes were proposed in this pull request?
   
   In ContainerManagerImpl, most of the methods accept a ContainerID object. This is then converted into `HddsProtos.ContainerID` to pass to ContainerStateManagerImpl. The data is stored in ContainerStateMap, where it is all keyed on ContainerID, so the `HddsProtos.ContainerID` has to be converted back to ContainerID again, which seems wasteful.
   
   There are some methods in the ContainerStateManager interface that are annotated "@replicate", and they need to continue to passed the ContainerID Protobuf, but they are few and can be handled as special cases.
   
   There are also some TODO's in the code suggesting it was always the intention to make this change, but it never happened, eg:
   
   ```
     @Override
     public boolean contains(final HddsProtos.ContainerID id) {
       lock.readLock().lock();
       try {
         // TODO: Remove the protobuf conversion after fixing ContainerStateMap.
         return containers.contains(ContainerID.getFromProtobuf(id));
       } finally {
         lock.readLock().unlock();
       }
     }
   ```
   
   ## What is the link to the Apache JIRA
   
   https://issues.apache.org/jira/browse/HDDS-6334
   
   ## How was this patch tested?
   
   Existing tests cover 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.

To unsubscribe, e-mail: issues-unsubscribe@ozone.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



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


[GitHub] [ozone] adoroszlai commented on a change in pull request #3110: HDDS-6334. Remove ContainerID to Proto to ContainerID conversion in ContainerStateManagerImpl

Posted by GitBox <gi...@apache.org>.
adoroszlai commented on a change in pull request #3110:
URL: https://github.com/apache/ozone/pull/3110#discussion_r811963462



##########
File path: hadoop-hdds/server-scm/src/main/java/org/apache/hadoop/hdds/scm/container/ContainerStateManagerImpl.java
##########
@@ -326,11 +326,11 @@ public void addContainer(final ContainerInfoProto containerInfo)
   }
 
   @Override
-  public boolean contains(final HddsProtos.ContainerID id) {
+  public boolean contains(ContainerID id) {
     lock.readLock().lock();
     try {
       // TODO: Remove the protobuf conversion after fixing ContainerStateMap.

Review comment:
       TODO: remove the TODO after addressing it. ;)

##########
File path: hadoop-hdds/server-scm/src/main/java/org/apache/hadoop/hdds/scm/container/ContainerManagerImpl.java
##########
@@ -278,18 +275,17 @@ private ContainerInfo allocateContainer(final Pipeline pipeline,
         .build();
     containerStateManager.addContainer(containerInfo);
     scmContainerManagerMetrics.incNumSuccessfulCreateContainers();
-    return containerStateManager.getContainer(containerID.getProtobuf());
+    return containerStateManager.getContainer(containerID);
   }
 
   @Override
-  public void updateContainerState(final ContainerID id,
+  public void updateContainerState(final ContainerID cid,
                                    final LifeCycleEvent event)
       throws IOException, InvalidStateTransitionException {
-    final HddsProtos.ContainerID cid = id.getProtobuf();
     lock.lock();
     try {
       if (containerExist(cid)) {
-        containerStateManager.updateContainerState(cid, event);
+        containerStateManager.updateContainerState(cid.getProtobuf(), event);

Review comment:
       Is it worth moving the `getProtobuf()` call inside the locked section, to avoid calls for nonexistent containers?

##########
File path: hadoop-hdds/server-scm/src/main/java/org/apache/hadoop/hdds/scm/container/ContainerManagerImpl.java
##########
@@ -414,13 +408,12 @@ public void notifyContainerReportProcessing(final boolean isFullReport,
   }
 
   @Override
-  public void deleteContainer(final ContainerID id)
+  public void deleteContainer(final ContainerID cid)
       throws IOException {
-    final HddsProtos.ContainerID cid = id.getProtobuf();
     lock.lock();
     try {
       if (containerExist(cid)) {
-        containerStateManager.removeContainer(cid);
+        containerStateManager.removeContainer(cid.getProtobuf());

Review comment:
       Same question here on lock vs. unnecessary call.




-- 
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.

To unsubscribe, e-mail: issues-unsubscribe@ozone.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



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


[GitHub] [ozone] sodonnel commented on a change in pull request #3110: HDDS-6334. Remove ContainerID to Proto to ContainerID conversion in ContainerStateManagerImpl

Posted by GitBox <gi...@apache.org>.
sodonnel commented on a change in pull request #3110:
URL: https://github.com/apache/ozone/pull/3110#discussion_r812368387



##########
File path: hadoop-hdds/server-scm/src/main/java/org/apache/hadoop/hdds/scm/container/ContainerManagerImpl.java
##########
@@ -278,18 +275,17 @@ private ContainerInfo allocateContainer(final Pipeline pipeline,
         .build();
     containerStateManager.addContainer(containerInfo);
     scmContainerManagerMetrics.incNumSuccessfulCreateContainers();
-    return containerStateManager.getContainer(containerID.getProtobuf());
+    return containerStateManager.getContainer(containerID);
   }
 
   @Override
-  public void updateContainerState(final ContainerID id,
+  public void updateContainerState(final ContainerID cid,
                                    final LifeCycleEvent event)
       throws IOException, InvalidStateTransitionException {
-    final HddsProtos.ContainerID cid = id.getProtobuf();
     lock.lock();
     try {
       if (containerExist(cid)) {
-        containerStateManager.updateContainerState(cid, event);
+        containerStateManager.updateContainerState(cid.getProtobuf(), event);

Review comment:
       I see your point. I'd guess the number of "not found" containers would be very low, so it probably makes sense to lift the protobuf overhead out of the lock.




-- 
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.

To unsubscribe, e-mail: issues-unsubscribe@ozone.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



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


[GitHub] [ozone] sodonnel commented on a change in pull request #3110: HDDS-6334. Remove ContainerID to Proto to ContainerID conversion in ContainerStateManagerImpl

Posted by GitBox <gi...@apache.org>.
sodonnel commented on a change in pull request #3110:
URL: https://github.com/apache/ozone/pull/3110#discussion_r812309476



##########
File path: hadoop-hdds/server-scm/src/main/java/org/apache/hadoop/hdds/scm/container/ContainerStateManagerImpl.java
##########
@@ -326,11 +326,11 @@ public void addContainer(final ContainerInfoProto containerInfo)
   }
 
   @Override
-  public boolean contains(final HddsProtos.ContainerID id) {
+  public boolean contains(ContainerID id) {
     lock.readLock().lock();
     try {
       // TODO: Remove the protobuf conversion after fixing ContainerStateMap.

Review comment:
       Good point!




-- 
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.

To unsubscribe, e-mail: issues-unsubscribe@ozone.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



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


[GitHub] [ozone] sodonnel merged pull request #3110: HDDS-6334. Remove ContainerID to Proto to ContainerID conversion in ContainerStateManagerImpl

Posted by GitBox <gi...@apache.org>.
sodonnel merged pull request #3110:
URL: https://github.com/apache/ozone/pull/3110


   


-- 
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.

To unsubscribe, e-mail: issues-unsubscribe@ozone.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



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


[GitHub] [ozone] sodonnel commented on a change in pull request #3110: HDDS-6334. Remove ContainerID to Proto to ContainerID conversion in ContainerStateManagerImpl

Posted by GitBox <gi...@apache.org>.
sodonnel commented on a change in pull request #3110:
URL: https://github.com/apache/ozone/pull/3110#discussion_r812308095



##########
File path: hadoop-hdds/server-scm/src/main/java/org/apache/hadoop/hdds/scm/container/ContainerManagerImpl.java
##########
@@ -414,13 +408,12 @@ public void notifyContainerReportProcessing(final boolean isFullReport,
   }
 
   @Override
-  public void deleteContainer(final ContainerID id)
+  public void deleteContainer(final ContainerID cid)
       throws IOException {
-    final HddsProtos.ContainerID cid = id.getProtobuf();
     lock.lock();
     try {
       if (containerExist(cid)) {
-        containerStateManager.removeContainer(cid);
+        containerStateManager.removeContainer(cid.getProtobuf());

Review comment:
       Same here - the `getProtobuf()` call is inside the the lock, but before it was outside 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.

To unsubscribe, e-mail: issues-unsubscribe@ozone.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



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


[GitHub] [ozone] sodonnel commented on a change in pull request #3110: HDDS-6334. Remove ContainerID to Proto to ContainerID conversion in ContainerStateManagerImpl

Posted by GitBox <gi...@apache.org>.
sodonnel commented on a change in pull request #3110:
URL: https://github.com/apache/ozone/pull/3110#discussion_r812307213



##########
File path: hadoop-hdds/server-scm/src/main/java/org/apache/hadoop/hdds/scm/container/ContainerManagerImpl.java
##########
@@ -278,18 +275,17 @@ private ContainerInfo allocateContainer(final Pipeline pipeline,
         .build();
     containerStateManager.addContainer(containerInfo);
     scmContainerManagerMetrics.incNumSuccessfulCreateContainers();
-    return containerStateManager.getContainer(containerID.getProtobuf());
+    return containerStateManager.getContainer(containerID);
   }
 
   @Override
-  public void updateContainerState(final ContainerID id,
+  public void updateContainerState(final ContainerID cid,
                                    final LifeCycleEvent event)
       throws IOException, InvalidStateTransitionException {
-    final HddsProtos.ContainerID cid = id.getProtobuf();
     lock.lock();
     try {
       if (containerExist(cid)) {
-        containerStateManager.updateContainerState(cid, event);
+        containerStateManager.updateContainerState(cid.getProtobuf(), event);

Review comment:
       For this one, the `getProtobuf()` call is already inside the locked section, or do I not understand the issue?




-- 
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.

To unsubscribe, e-mail: issues-unsubscribe@ozone.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



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


[GitHub] [ozone] adoroszlai commented on a change in pull request #3110: HDDS-6334. Remove ContainerID to Proto to ContainerID conversion in ContainerStateManagerImpl

Posted by GitBox <gi...@apache.org>.
adoroszlai commented on a change in pull request #3110:
URL: https://github.com/apache/ozone/pull/3110#discussion_r812363115



##########
File path: hadoop-hdds/server-scm/src/main/java/org/apache/hadoop/hdds/scm/container/ContainerManagerImpl.java
##########
@@ -278,18 +275,17 @@ private ContainerInfo allocateContainer(final Pipeline pipeline,
         .build();
     containerStateManager.addContainer(containerInfo);
     scmContainerManagerMetrics.incNumSuccessfulCreateContainers();
-    return containerStateManager.getContainer(containerID.getProtobuf());
+    return containerStateManager.getContainer(containerID);
   }
 
   @Override
-  public void updateContainerState(final ContainerID id,
+  public void updateContainerState(final ContainerID cid,
                                    final LifeCycleEvent event)
       throws IOException, InvalidStateTransitionException {
-    final HddsProtos.ContainerID cid = id.getProtobuf();
     lock.lock();
     try {
       if (containerExist(cid)) {
-        containerStateManager.updateContainerState(cid, event);
+        containerStateManager.updateContainerState(cid.getProtobuf(), event);

Review comment:
       > For this one, the `getProtobuf()` call is already inside the locked section,
   
   Yes, that's what I'd like toavoid if possible, since container ID need not be guarded by the lock during conversion.  I was wondering about the pros/cons:
   
    * outside the lock: called eagerly (even if container does not exist), but does not hold lock
    * inside lock: may avoid some unnecessary calls, but increases the time lock is held




-- 
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.

To unsubscribe, e-mail: issues-unsubscribe@ozone.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



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