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 2021/08/04 17:53:41 UTC

[GitHub] [ozone] avijayanhwx commented on a change in pull request #2489: HDDS-5449 Recon namespace summary 'du' information should return replicated size of a key

avijayanhwx commented on a change in pull request #2489:
URL: https://github.com/apache/ozone/pull/2489#discussion_r682832116



##########
File path: hadoop-ozone/recon/src/main/java/org/apache/hadoop/ozone/recon/api/NSSummaryEndpoint.java
##########
@@ -267,26 +368,69 @@ public Response getDiskUsage(@QueryParam("path") String path)
         // reformat the response
         diskUsage.setSubpath(subpath);
         long dataSize = getTotalSize(subdirObjectId);
+        dirDataSize += dataSize;
+
+        if (withReplica) {
+          long subdirDU = 0;
+          List<OmKeyInfo> keys = listKeysUnderDirectory(subdirObjectId);
+          for (OmKeyInfo keyInfo: keys) {
+            subdirDU += getKeySizeWithReplication(keyInfo);
+          }
+          diskUsage.setSizeWithReplica(subdirDU);
+          dirDataSizeWithReplica += subdirDU;
+        }
+
         diskUsage.setSize(dataSize);
         subdirDUData.add(diskUsage);
       }
+
+      // handle direct keys under directory
+      if (listFile || withReplica) {
+        List<OmKeyInfo> directKeys = listDirectKeys(dirObjectId);
+
+        for (OmKeyInfo directKey: directKeys) {
+          DUResponse.DiskUsage diskUsage = new DUResponse.DiskUsage();
+          String subpath = buildSubpath(normalizedPath,
+                  directKey.getFileName());
+          diskUsage.setSubpath(subpath);
+          diskUsage.setSize(directKey.getDataSize());
+
+          if (withReplica) {
+            long keyDU = getKeySizeWithReplication(directKey);
+            dirDataSizeWithReplica += keyDU;
+            diskUsage.setSizeWithReplica(keyDU);
+          }
+          // list the key as a subpath
+          if (listFile) {
+            subdirDUData.add(diskUsage);
+          }
+        }
+      }
+
+      if (withReplica) {
+        duResponse.setSizeWithReplica(dirDataSizeWithReplica);
+      }
+      duResponse.setCount(subdirDUData.size());
+      duResponse.setSize(dirDataSize);
       duResponse.setDuData(subdirDUData);
       break;
     case KEY:
-      // DU for key is the data size
-      duResponse.setCount(1);
-      DUResponse.DiskUsage keyDU = new DUResponse.DiskUsage();
+      // DU for key doesn't have subpaths
+      duResponse.setCount(0);
       // The object ID for the directory that the key is directly in
       long parentObjectId = getDirObjectId(names, names.length - 1);
       String fileName = names[names.length - 1];
       String ozoneKey =
               omMetadataManager.getOzonePathKey(parentObjectId, fileName);
       OmKeyInfo keyInfo =
               omMetadataManager.getFileTable().getSkipCache(ozoneKey);
-      String subpath = buildSubpath(normalizedPath, null);
-      keyDU.setSubpath(subpath);
-      keyDU.setSize(keyInfo.getDataSize());
-      duResponse.setDuData(Collections.singletonList(keyDU));
+      duResponse.setSize(keyInfo.getDataSize());
+      if (withReplica) {
+        long keySizeWithReplica = getKeySizeWithReplication(keyInfo);
+        duResponse.setSizeWithReplica(keySizeWithReplica);
+      }
+      // give an empty list to avoid NPE
+      duResponse.setDuData(new ArrayList<>());

Review comment:
       We could init the duData list to empty list in the DuResponse constructor.

##########
File path: hadoop-ozone/recon/src/main/java/org/apache/hadoop/ozone/recon/api/NSSummaryEndpoint.java
##########
@@ -724,6 +868,135 @@ private long getTotalSize(long objectId) throws IOException {
     return result;
   }
 
+  private List<OmKeyInfo> listKeysUnderVolume(String volumeName)
+          throws IOException {
+    List<OmKeyInfo> result = new ArrayList<>();
+    if (Strings.isBlank(volumeName)) {
+      return result;
+    }
+
+    Table keyTable = omMetadataManager.getFileTable();
+
+    TableIterator<String, ? extends Table.KeyValue<String, OmKeyInfo>>
+            iterator = keyTable.iterator();
+
+    while (iterator.hasNext()) {
+      Table.KeyValue<String, OmKeyInfo> kv = iterator.next();
+      OmKeyInfo keyInfo = kv.getValue();
+
+      if (keyInfo != null) {
+        if (volumeName.equals(keyInfo.getVolumeName())) {
+          result.add(keyInfo);
+        }
+      }
+    }
+    return result;
+  }
+
+  private List<OmKeyInfo> listKeysUnderBucket(String bucketName)
+          throws IOException {
+    List<OmKeyInfo> result = new ArrayList<>();
+    if (Strings.isBlank(bucketName)) {
+      return result;
+    }
+
+    Table keyTable = omMetadataManager.getFileTable();
+
+    TableIterator<String, ? extends Table.KeyValue<String, OmKeyInfo>>
+            iterator = keyTable.iterator();
+
+    while (iterator.hasNext()) {
+      Table.KeyValue<String, OmKeyInfo> kv = iterator.next();
+      OmKeyInfo keyInfo = kv.getValue();
+
+      if (keyInfo != null) {
+        if (bucketName.equals(keyInfo.getBucketName())) {
+          result.add(keyInfo);
+        }
+      }
+    }
+    return result;
+  }
+
+  private List<OmKeyInfo> listKeysUnderDirectory(long dirObjectId)
+          throws IOException {
+    List<OmKeyInfo> result = new ArrayList<>();
+
+    Table keyTable = omMetadataManager.getFileTable();
+
+    TableIterator<String, ? extends Table.KeyValue<String, OmKeyInfo>>
+            iterator = keyTable.iterator();
+
+    while (iterator.hasNext()) {
+      Table.KeyValue<String, OmKeyInfo> kv = iterator.next();
+      OmKeyInfo keyInfo = kv.getValue();
+
+      if (keyInfo != null
+              && isFileUnderDir(dirObjectId, keyInfo.getParentObjectID())) {
+        result.add(keyInfo);
+      }
+    }
+    return result;
+  }
+
+  private boolean isFileUnderDir(long dirObjectId,
+                                 long keyParentId) throws IOException {
+    if (dirObjectId == keyParentId) {
+      return true;
+    } else {
+      NSSummary nsSummary =
+              reconNamespaceSummaryManager.getNSSummary(dirObjectId);
+      for (long subdirId: nsSummary.getChildDir()) {
+        if (isFileUnderDir(subdirId, keyParentId)) {
+          return true;
+        }
+      }
+      return false;
+    }
+  }
+
+  private List<OmKeyInfo> listDirectKeys(long parentId) throws IOException {
+    List<OmKeyInfo> result = new ArrayList<>();
+
+    Table keyTable = omMetadataManager.getFileTable();
+
+    TableIterator<String, ? extends Table.KeyValue<String, OmKeyInfo>>
+            iterator = keyTable.iterator();
+
+    while (iterator.hasNext()) {
+      Table.KeyValue<String, OmKeyInfo> kv = iterator.next();
+      OmKeyInfo keyInfo = kv.getValue();
+
+      if (keyInfo != null) {
+        if (parentId == keyInfo.getParentObjectID()) {
+          result.add(keyInfo);
+        }
+      }
+    }
+    return result;
+  }
+
+  private long getKeySizeWithReplication(OmKeyInfo keyInfo) {
+    OmKeyLocationInfoGroup locationGroup = keyInfo.getLatestVersionLocations();
+    List<OmKeyLocationInfo> keyLocations =
+            locationGroup.getBlocksLatestVersionOnly();
+    long du = keyInfo.getDataSize();
+    // a key could be too large to fit in one single container
+    for (OmKeyLocationInfo location: keyLocations) {
+      BlockID block = location.getBlockID();
+      ContainerID containerId = new ContainerID(block.getContainerID());
+      try {
+        int replicationFactor =
+                containerManager.getContainerReplicas(containerId).size();
+        du *= replicationFactor;
+        break;
+      } catch (ContainerNotFoundException cnfe) {
+        LOG.warn("Cannot find container");

Review comment:
       Can be debug, along with the ContainerID being printed out.

##########
File path: hadoop-ozone/recon/src/main/java/org/apache/hadoop/ozone/recon/api/NSSummaryEndpoint.java
##########
@@ -724,6 +868,135 @@ private long getTotalSize(long objectId) throws IOException {
     return result;
   }
 
+  private List<OmKeyInfo> listKeysUnderVolume(String volumeName)
+          throws IOException {
+    List<OmKeyInfo> result = new ArrayList<>();
+    if (Strings.isBlank(volumeName)) {
+      return result;
+    }
+
+    Table keyTable = omMetadataManager.getFileTable();
+
+    TableIterator<String, ? extends Table.KeyValue<String, OmKeyInfo>>
+            iterator = keyTable.iterator();
+
+    while (iterator.hasNext()) {
+      Table.KeyValue<String, OmKeyInfo> kv = iterator.next();

Review comment:
       Can we use the iterator.seek() API that let's us jump to the first key with that the volume name in prefix here?

##########
File path: hadoop-ozone/recon/src/main/java/org/apache/hadoop/ozone/recon/api/NSSummaryEndpoint.java
##########
@@ -724,6 +868,135 @@ private long getTotalSize(long objectId) throws IOException {
     return result;
   }
 
+  private List<OmKeyInfo> listKeysUnderVolume(String volumeName)
+          throws IOException {
+    List<OmKeyInfo> result = new ArrayList<>();
+    if (Strings.isBlank(volumeName)) {
+      return result;
+    }
+
+    Table keyTable = omMetadataManager.getFileTable();
+
+    TableIterator<String, ? extends Table.KeyValue<String, OmKeyInfo>>
+            iterator = keyTable.iterator();
+
+    while (iterator.hasNext()) {
+      Table.KeyValue<String, OmKeyInfo> kv = iterator.next();
+      OmKeyInfo keyInfo = kv.getValue();
+
+      if (keyInfo != null) {
+        if (volumeName.equals(keyInfo.getVolumeName())) {
+          result.add(keyInfo);
+        }
+      }
+    }
+    return result;
+  }
+
+  private List<OmKeyInfo> listKeysUnderBucket(String bucketName)
+          throws IOException {
+    List<OmKeyInfo> result = new ArrayList<>();
+    if (Strings.isBlank(bucketName)) {
+      return result;
+    }
+
+    Table keyTable = omMetadataManager.getFileTable();
+
+    TableIterator<String, ? extends Table.KeyValue<String, OmKeyInfo>>
+            iterator = keyTable.iterator();
+
+    while (iterator.hasNext()) {
+      Table.KeyValue<String, OmKeyInfo> kv = iterator.next();

Review comment:
       Same as above.

##########
File path: hadoop-ozone/recon/src/main/java/org/apache/hadoop/ozone/recon/api/NSSummaryEndpoint.java
##########
@@ -724,6 +868,135 @@ private long getTotalSize(long objectId) throws IOException {
     return result;
   }
 
+  private List<OmKeyInfo> listKeysUnderVolume(String volumeName)
+          throws IOException {
+    List<OmKeyInfo> result = new ArrayList<>();
+    if (Strings.isBlank(volumeName)) {
+      return result;
+    }
+
+    Table keyTable = omMetadataManager.getFileTable();
+
+    TableIterator<String, ? extends Table.KeyValue<String, OmKeyInfo>>
+            iterator = keyTable.iterator();
+
+    while (iterator.hasNext()) {
+      Table.KeyValue<String, OmKeyInfo> kv = iterator.next();
+      OmKeyInfo keyInfo = kv.getValue();
+
+      if (keyInfo != null) {
+        if (volumeName.equals(keyInfo.getVolumeName())) {
+          result.add(keyInfo);
+        }
+      }
+    }
+    return result;
+  }
+
+  private List<OmKeyInfo> listKeysUnderBucket(String bucketName)
+          throws IOException {
+    List<OmKeyInfo> result = new ArrayList<>();
+    if (Strings.isBlank(bucketName)) {
+      return result;
+    }
+
+    Table keyTable = omMetadataManager.getFileTable();
+
+    TableIterator<String, ? extends Table.KeyValue<String, OmKeyInfo>>
+            iterator = keyTable.iterator();
+
+    while (iterator.hasNext()) {
+      Table.KeyValue<String, OmKeyInfo> kv = iterator.next();
+      OmKeyInfo keyInfo = kv.getValue();
+
+      if (keyInfo != null) {
+        if (bucketName.equals(keyInfo.getBucketName())) {
+          result.add(keyInfo);
+        }
+      }
+    }
+    return result;
+  }
+
+  private List<OmKeyInfo> listKeysUnderDirectory(long dirObjectId)
+          throws IOException {
+    List<OmKeyInfo> result = new ArrayList<>();
+
+    Table keyTable = omMetadataManager.getFileTable();
+
+    TableIterator<String, ? extends Table.KeyValue<String, OmKeyInfo>>
+            iterator = keyTable.iterator();
+
+    while (iterator.hasNext()) {

Review comment:
       Same as above.

##########
File path: hadoop-ozone/recon/src/test/java/org/apache/hadoop/ozone/recon/api/TestNSSummaryEndpoint.java
##########
@@ -299,13 +311,15 @@ public void testDiskUsage() throws Exception {
     Assert.assertEquals(KEY_SIX_SIZE, duDir4.getSize());
 
     // key level DU
-    Response keyResponse = nsSummaryEndpoint.getDiskUsage(KEY_PATH);
+    Response keyResponse = nsSummaryEndpoint.getDiskUsage(KEY_PATH,
+            false, false);

Review comment:
       Can we add tests for replica size? There are a few cases that need to be verified (key spanning into multiple blocks, container replica count < desired replica count etc) 

##########
File path: hadoop-ozone/recon/src/main/java/org/apache/hadoop/ozone/recon/api/NSSummaryEndpoint.java
##########
@@ -62,17 +72,25 @@
 @Path("/namespace")
 @Produces(MediaType.APPLICATION_JSON)
 public class NSSummaryEndpoint {
+
+  private static final Logger LOG = LoggerFactory.getLogger(
+          NSSummaryEndpoint.class);
   @Inject
   private ReconNamespaceSummaryManager reconNamespaceSummaryManager;
 
   @Inject
   private ReconOMMetadataManager omMetadataManager;
 
+  private ReconContainerManager containerManager;
+
   @Inject
   public NSSummaryEndpoint(ReconNamespaceSummaryManager namespaceSummaryManager,
-                           ReconOMMetadataManager omMetadataManager) {
+                           ReconOMMetadataManager omMetadataManager,
+                           OzoneStorageContainerManager reconSCM) {
     this.reconNamespaceSummaryManager = namespaceSummaryManager;
     this.omMetadataManager = omMetadataManager;
+    this.containerManager =
+            (ReconContainerManager) reconSCM.getContainerManager();

Review comment:
       We don't need the cast to RCM here. We should be OK with the return type being 'ContainerManagerV2' since we are using only interface method.

##########
File path: hadoop-ozone/recon/src/main/java/org/apache/hadoop/ozone/recon/api/NSSummaryEndpoint.java
##########
@@ -163,12 +181,19 @@ public Response getBasicInfo(
   /**
    * DU endpoint to return datasize for subdirectory (bucket for volume).
    * @param path request path
+   * @param listFile show subpath/disk usage for each key
+   * @param withReplica count actual DU with replication
    * @return DU response
    * @throws IOException
    */
   @GET
   @Path("/du")
-  public Response getDiskUsage(@QueryParam("path") String path)
+  @SuppressWarnings("methodlength")
+  public Response getDiskUsage(@QueryParam("path") String path,
+                               @DefaultValue("false")
+                               @QueryParam("files") boolean listFile,
+                               @DefaultValue("false")
+                               @QueryParam("replica") boolean withReplica)

Review comment:
       Can we set Default value for replica?




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