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

guohao1225 pushed a commit to branch guohao-HDDS-10767-dev
in repository https://gitbox.apache.org/repos/asf/ozone.git

commit 827be5f29a5927bbb4eb46a9a42b5a19f28924f6
Author: guohao1 <[email protected]>
AuthorDate: Sun Apr 28 15:57:10 2024 +0800

    HDDS-10767. Reducing DatanodeDetails in the ContainerLocationCache
---
 .../java/org/apache/hadoop/ozone/om/ScmClient.java | 33 ++++++++++++++++++++--
 .../org/apache/hadoop/ozone/om/TestScmClient.java  | 15 ++++++----
 2 files changed, 39 insertions(+), 9 deletions(-)

diff --git 
a/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/ScmClient.java
 
b/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/ScmClient.java
index 77ee0d5851..b5a12dd0c3 100644
--- 
a/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/ScmClient.java
+++ 
b/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/ScmClient.java
@@ -22,7 +22,7 @@ import com.google.common.cache.CacheLoader;
 import com.google.common.cache.CacheLoader.InvalidCacheLoadException;
 import com.google.common.cache.LoadingCache;
 import org.apache.hadoop.hdds.conf.OzoneConfiguration;
-import 
org.apache.hadoop.hdds.scm.container.common.helpers.ContainerWithPipeline;
+import org.apache.hadoop.hdds.protocol.DatanodeDetails;
 import org.apache.hadoop.hdds.scm.pipeline.Pipeline;
 import org.apache.hadoop.hdds.scm.protocol.ScmBlockLocationProtocol;
 import org.apache.hadoop.hdds.scm.protocol.StorageContainerLocationProtocol;
@@ -30,9 +30,12 @@ import org.apache.hadoop.util.CacheMetrics;
 import jakarta.annotation.Nonnull;
 
 import java.io.IOException;
+import java.util.ArrayList;
 import java.util.HashMap;
 import java.util.List;
 import java.util.Map;
+import java.util.UUID;
+import java.util.concurrent.ConcurrentHashMap;
 import java.util.concurrent.ExecutionException;
 import java.util.concurrent.TimeUnit;
 import java.util.stream.Collectors;
@@ -72,6 +75,9 @@ public class ScmClient {
     long ttl = configuration.getTimeDuration(
         OZONE_OM_CONTAINER_LOCATION_CACHE_TTL,
         OZONE_OM_CONTAINER_LOCATION_CACHE_TTL_DEFAULT.getDuration(), unit);
+
+    final Map<UUID, DatanodeDetails>
+        datanodeDetailsCache = new ConcurrentHashMap<>();
     return CacheBuilder.newBuilder()
         .maximumSize(maxSize)
         .expireAfterWrite(ttl, unit)
@@ -80,7 +86,9 @@ public class ScmClient {
           @Nonnull
           @Override
           public Pipeline load(@Nonnull Long key) throws Exception {
-            return containerClient.getContainerWithPipeline(key).getPipeline();
+            Pipeline pipeline =
+                containerClient.getContainerWithPipeline(key).getPipeline();
+            return newPipelineWithDNCache(pipeline, datanodeDetailsCache);
           }
 
           @Nonnull
@@ -91,12 +99,31 @@ public class ScmClient {
                 .stream()
                 .collect(Collectors.toMap(
                     x -> x.getContainerInfo().getContainerID(),
-                    ContainerWithPipeline::getPipeline
+                    x -> newPipelineWithDNCache(x.getPipeline(),
+                        datanodeDetailsCache)
                 ));
           }
         });
   }
 
+  static Pipeline newPipelineWithDNCache(Pipeline pipeline,
+      Map<UUID, DatanodeDetails> datanodeDetailsCache) {
+    Pipeline.Builder builder = Pipeline.newBuilder(pipeline);
+    List<DatanodeDetails> nodes = new ArrayList<>();
+    for (DatanodeDetails node : pipeline.getNodes()) {
+      DatanodeDetails datanodeDetails =
+          datanodeDetailsCache.get(node.getUuid());
+      if (node.equals(datanodeDetails)) {
+        nodes.add(datanodeDetails);
+      } else {
+        datanodeDetailsCache.put(node.getUuid(), node);
+        nodes.add(node);
+      }
+    }
+    builder.setNodes(nodes);
+    return builder.build();
+  }
+
   public ScmBlockLocationProtocol getBlockClient() {
     return this.blockClient;
   }
diff --git 
a/hadoop-ozone/ozone-manager/src/test/java/org/apache/hadoop/ozone/om/TestScmClient.java
 
b/hadoop-ozone/ozone-manager/src/test/java/org/apache/hadoop/ozone/om/TestScmClient.java
index 228f668d01..4926bbbdc4 100644
--- 
a/hadoop-ozone/ozone-manager/src/test/java/org/apache/hadoop/ozone/om/TestScmClient.java
+++ 
b/hadoop-ozone/ozone-manager/src/test/java/org/apache/hadoop/ozone/om/TestScmClient.java
@@ -42,7 +42,6 @@ import java.util.UUID;
 import java.util.stream.Stream;
 
 import static com.google.common.collect.Sets.newHashSet;
-import static java.util.Arrays.asList;
 import static org.apache.commons.lang.RandomStringUtils.randomAlphabetic;
 import static 
org.apache.hadoop.hdds.client.ReplicationConfig.fromTypeAndFactor;
 import static org.junit.jupiter.api.Assertions.assertEquals;
@@ -107,9 +106,12 @@ public class TestScmClient {
       throws IOException {
 
     Map<Long, ContainerWithPipeline> actualLocations = new HashMap<>();
-
+    List<DatanodeDetails> dnList = new ArrayList<>();
+    for (int i = 0; i < 3; i++) {
+      dnList.add(randomDatanode());
+    }
     for (long containerId : prepopulatedIds) {
-      ContainerWithPipeline pipeline = createPipeline(containerId);
+      ContainerWithPipeline pipeline = createPipeline(containerId, dnList);
       actualLocations.put(containerId, pipeline);
     }
 
@@ -129,7 +131,7 @@ public class TestScmClient {
     if (!expectedScmCallIds.isEmpty()) {
       List<ContainerWithPipeline> scmLocations = new ArrayList<>();
       for (long containerId : expectedScmCallIds) {
-        ContainerWithPipeline pipeline = createPipeline(containerId);
+        ContainerWithPipeline pipeline = createPipeline(containerId, dnList);
         scmLocations.add(pipeline);
         actualLocations.put(containerId, pipeline);
       }
@@ -167,13 +169,14 @@ public class TestScmClient {
     assertEquals(runtimeException, actualRt.getCause());
   }
 
-  ContainerWithPipeline createPipeline(long containerId) {
+  ContainerWithPipeline createPipeline(long containerId,
+                                       List<DatanodeDetails> dnList) {
     ContainerInfo containerInfo = new ContainerInfo.Builder()
         .setContainerID(containerId)
         .build();
     Pipeline pipeline = Pipeline.newBuilder()
         .setId(PipelineID.randomId())
-        .setNodes(asList(randomDatanode(), randomDatanode()))
+        .setNodes(dnList)
         .setReplicationConfig(fromTypeAndFactor(
             ReplicationType.RATIS, ReplicationFactor.THREE))
         .setState(Pipeline.PipelineState.OPEN)


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

Reply via email to