This is an automated email from the ASF dual-hosted git repository.

siddhant pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/ozone.git


The following commit(s) were added to refs/heads/master by this push:
     new 0697045a59 HDDS-10870. moveSelectionToFutureMap cleanup when future 
complete (#6746)
0697045a59 is described below

commit 0697045a5993d5c834f90ca11161f1019c389705
Author: hao guo <[email protected]>
AuthorDate: Mon Jun 10 16:56:09 2024 +0800

    HDDS-10870. moveSelectionToFutureMap cleanup when future complete (#6746)
---
 .../container/balancer/ContainerBalancerTask.java  | 40 +++++++++++++---------
 .../container/balancer/ContainerMoveSelection.java | 22 ++++++++++++
 2 files changed, 45 insertions(+), 17 deletions(-)

diff --git 
a/hadoop-hdds/server-scm/src/main/java/org/apache/hadoop/hdds/scm/container/balancer/ContainerBalancerTask.java
 
b/hadoop-hdds/server-scm/src/main/java/org/apache/hadoop/hdds/scm/container/balancer/ContainerBalancerTask.java
index a4072c8d4f..8daf764057 100644
--- 
a/hadoop-hdds/server-scm/src/main/java/org/apache/hadoop/hdds/scm/container/balancer/ContainerBalancerTask.java
+++ 
b/hadoop-hdds/server-scm/src/main/java/org/apache/hadoop/hdds/scm/container/balancer/ContainerBalancerTask.java
@@ -44,6 +44,7 @@ import org.slf4j.LoggerFactory;
 import java.io.IOException;
 import java.time.Duration;
 import java.util.ArrayList;
+import java.util.Collection;
 import java.util.Collections;
 import java.util.HashMap;
 import java.util.HashSet;
@@ -52,6 +53,7 @@ import java.util.List;
 import java.util.Map;
 import java.util.Set;
 import java.util.concurrent.CompletableFuture;
+import java.util.concurrent.ConcurrentHashMap;
 import java.util.concurrent.ExecutionException;
 import java.util.concurrent.TimeUnit;
 import java.util.concurrent.TimeoutException;
@@ -473,7 +475,7 @@ public class ContainerBalancerTask implements Runnable {
     findTargetStrategy.reInitialize(potentialTargets, config, upperLimit);
     findSourceStrategy.reInitialize(getPotentialSources(), config, lowerLimit);
 
-    moveSelectionToFutureMap = new HashMap<>(underUtilizedNodes.size() + 
overUtilizedNodes.size());
+    moveSelectionToFutureMap = new ConcurrentHashMap<>();
     boolean isMoveGeneratedInThisIteration = false;
     iterationResult = IterationResult.ITERATION_COMPLETED;
     boolean canAdaptWhenNearingLimits = true;
@@ -599,21 +601,25 @@ public class ContainerBalancerTask implements Runnable {
    */
   private void checkIterationMoveResults() {
     this.countDatanodesInvolvedPerIteration = 0;
-    CompletableFuture<Void> allFuturesResult = CompletableFuture.allOf(
-        moveSelectionToFutureMap.values()
-            .toArray(new CompletableFuture[moveSelectionToFutureMap.size()]));
-    try {
-      allFuturesResult.get(config.getMoveTimeout().toMillis(),
-          TimeUnit.MILLISECONDS);
-    } catch (InterruptedException e) {
-      LOG.warn("Container balancer is interrupted");
-      Thread.currentThread().interrupt();
-    } catch (TimeoutException e) {
-      long timeoutCounts = cancelMovesThatExceedTimeoutDuration();
-      LOG.warn("{} Container moves are canceled.", timeoutCounts);
-      
metrics.incrementNumContainerMovesTimeoutInLatestIteration(timeoutCounts);
-    } catch (ExecutionException e) {
-      LOG.error("Got exception while checkIterationMoveResults", e);
+    Collection<CompletableFuture<MoveManager.MoveResult>> futures =
+        moveSelectionToFutureMap.values();
+    if (!futures.isEmpty()) {
+      CompletableFuture<Void> allFuturesResult = CompletableFuture.allOf(
+          futures.toArray(new CompletableFuture[futures.size()]));
+      try {
+        allFuturesResult.get(config.getMoveTimeout().toMillis(),
+            TimeUnit.MILLISECONDS);
+      } catch (InterruptedException e) {
+        LOG.warn("Container balancer is interrupted");
+        Thread.currentThread().interrupt();
+      } catch (TimeoutException e) {
+        long timeoutCounts = cancelMovesThatExceedTimeoutDuration();
+        LOG.warn("{} Container moves are canceled.", timeoutCounts);
+        metrics.incrementNumContainerMovesTimeoutInLatestIteration(
+            timeoutCounts);
+      } catch (ExecutionException e) {
+        LOG.error("Got exception while checkIterationMoveResults", e);
+      }
     }
 
     countDatanodesInvolvedPerIteration =
@@ -829,6 +835,7 @@ public class ContainerBalancerTask implements Runnable {
 
       future = future.whenComplete((result, ex) -> {
         metrics.incrementCurrentIterationContainerMoveMetric(result, 1);
+        moveSelectionToFutureMap.remove(moveSelection);
         if (ex != null) {
           LOG.info("Container move for container {} from source {} to " +
                   "target {} failed with exceptions.",
@@ -884,7 +891,6 @@ public class ContainerBalancerTask implements Runnable {
         return false;
       } else {
         MoveManager.MoveResult result = future.join();
-        moveSelectionToFutureMap.put(moveSelection, future);
         if (result == 
MoveManager.MoveResult.REPLICATION_FAIL_NOT_EXIST_IN_SOURCE ||
             result == MoveManager.MoveResult.REPLICATION_FAIL_EXIST_IN_TARGET 
||
             result == 
MoveManager.MoveResult.REPLICATION_FAIL_CONTAINER_NOT_CLOSED ||
diff --git 
a/hadoop-hdds/server-scm/src/main/java/org/apache/hadoop/hdds/scm/container/balancer/ContainerMoveSelection.java
 
b/hadoop-hdds/server-scm/src/main/java/org/apache/hadoop/hdds/scm/container/balancer/ContainerMoveSelection.java
index 1a8cbee09d..118fa78e32 100644
--- 
a/hadoop-hdds/server-scm/src/main/java/org/apache/hadoop/hdds/scm/container/balancer/ContainerMoveSelection.java
+++ 
b/hadoop-hdds/server-scm/src/main/java/org/apache/hadoop/hdds/scm/container/balancer/ContainerMoveSelection.java
@@ -20,6 +20,8 @@ package org.apache.hadoop.hdds.scm.container.balancer;
 import org.apache.hadoop.hdds.protocol.DatanodeDetails;
 import org.apache.hadoop.hdds.scm.container.ContainerID;
 
+import java.util.Objects;
+
 /**
  * This class represents a target datanode and the container to be moved from
  * a source to that target.
@@ -52,4 +54,24 @@ public class ContainerMoveSelection {
       ContainerID containerID) {
     this.containerID = containerID;
   }
+
+  @Override
+  public boolean equals(Object o) {
+    if (this == o) {
+      return true;
+    }
+    if (o == null || getClass() != o.getClass()) {
+      return false;
+    }
+    ContainerMoveSelection that = (ContainerMoveSelection) o;
+    if (targetNode != that.targetNode) {
+      return false;
+    }
+    return containerID == that.containerID;
+  }
+
+  @Override
+  public int hashCode() {
+    return Objects.hash(targetNode, containerID);
+  }
 }


---------------------------------------------------------------------
To unsubscribe, e-mail: [email protected]
For additional commands, e-mail: [email protected]

Reply via email to