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/09/24 01:57:03 UTC

[GitHub] [ozone] GeorgeJahad commented on a diff in pull request #3746: HDDS-7121. Support namespace summaries (du, dist & counts) for legacy FS buckets

GeorgeJahad commented on code in PR #3746:
URL: https://github.com/apache/ozone/pull/3746#discussion_r979138926


##########
hadoop-ozone/recon/src/main/java/org/apache/hadoop/ozone/recon/api/handlers/LegacyBucketHandler.java:
##########
@@ -0,0 +1,317 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.hadoop.ozone.recon.api.handlers;
+
+import org.apache.hadoop.hdds.scm.server.OzoneStorageContainerManager;
+import org.apache.hadoop.hdds.utils.db.Table;
+import org.apache.hadoop.hdds.utils.db.TableIterator;
+import org.apache.hadoop.ozone.om.helpers.BucketLayout;
+import org.apache.hadoop.ozone.om.helpers.OmBucketInfo;
+import org.apache.hadoop.ozone.om.helpers.OmKeyInfo;
+import org.apache.hadoop.ozone.om.helpers.OzoneFSUtils;
+import org.apache.hadoop.ozone.recon.api.types.DUResponse;
+import org.apache.hadoop.ozone.recon.api.types.EntityType;
+import org.apache.hadoop.ozone.recon.api.types.NSSummary;
+import org.apache.hadoop.ozone.recon.recovery.ReconOMMetadataManager;
+import org.apache.hadoop.ozone.recon.spi.ReconNamespaceSummaryManager;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import java.io.IOException;
+import java.util.List;
+import java.util.Set;
+
+import static org.apache.hadoop.ozone.OzoneConsts.OM_KEY_PREFIX;
+
+/**
+ * Class for handling Legacy buckets.
+ */
+public class LegacyBucketHandler extends BucketHandler {
+
+  private static final Logger LOG = LoggerFactory.getLogger(
+      LegacyBucketHandler.class);
+
+  private final String vol;
+  private final String bucket;
+  private final OmBucketInfo omBucketInfo;
+
+  public LegacyBucketHandler(
+      ReconNamespaceSummaryManager reconNamespaceSummaryManager,
+      ReconOMMetadataManager omMetadataManager,
+      OzoneStorageContainerManager reconSCM,
+      OmBucketInfo bucketInfo) {
+    super(reconNamespaceSummaryManager, omMetadataManager,
+        reconSCM);
+    this.omBucketInfo = bucketInfo;
+    this.vol = omBucketInfo.getVolumeName();
+    this.bucket = omBucketInfo.getBucketName();
+  }
+
+  /**
+   * Helper function to check if a path is a directory, key, or invalid.
+   * @param keyName key name
+   * @return DIRECTORY, KEY, or UNKNOWN
+   * @throws IOException
+   */
+  @Override
+  public EntityType determineKeyPath(String keyName)
+      throws IOException {
+
+    String filename = OzoneFSUtils.removeTrailingSlashIfNeeded(keyName);
+    // For example, /vol1/buck1/a/b/c/d/e/file1.txt
+    // Look in the KeyTable for the key path,
+    // if the first one we seek to is the same as the seek key,
+    // it is a key;
+    // if it is the seekKey with a trailing slash, it is a directory
+    // else it is unknown
+    String key = OM_KEY_PREFIX + vol +
+        OM_KEY_PREFIX + bucket +
+        OM_KEY_PREFIX + filename;
+
+    Table<String, OmKeyInfo> keyTable = getKeyTable();
+
+    TableIterator<String, ? extends Table.KeyValue<String, OmKeyInfo>>
+        iterator = keyTable.iterator();
+
+    iterator.seek(key);
+    if (iterator.hasNext()) {
+      Table.KeyValue<String, OmKeyInfo> kv = iterator.next();
+      String dbKey = kv.getKey();
+      if (dbKey.equals(key)) {
+        return EntityType.KEY;
+      }
+      if (dbKey.equals(key + OM_KEY_PREFIX)) {
+        return EntityType.DIRECTORY;
+      }
+    }
+    return EntityType.UNKNOWN;
+  }
+
+  // KeyTable's key is in the format of "vol/bucket/keyName"
+  // Make use of RocksDB's order to seek to the prefix and avoid full iteration
+  @Override
+  public long calculateDUUnderObject(long parentId)
+      throws IOException {
+    Table<String, OmKeyInfo> keyTable = getKeyTable();
+
+    long totalDU = 0L;
+    TableIterator<String, ? extends Table.KeyValue<String, OmKeyInfo>>
+        iterator = keyTable.iterator();
+
+    String seekPrefix = OM_KEY_PREFIX +
+        vol +
+        OM_KEY_PREFIX +
+        bucket +
+        OM_KEY_PREFIX;
+
+    NSSummary nsSummary = getReconNamespaceSummaryManager()
+        .getNSSummary(parentId);
+    // empty bucket
+    if (nsSummary == null) {
+      return 0;
+    }
+
+    if (omBucketInfo.getObjectID() != parentId) {
+      String dirName = nsSummary.getDirName();
+      seekPrefix += dirName;
+    }
+
+    String[] seekKeys = seekPrefix.split(OM_KEY_PREFIX);
+    iterator.seek(seekPrefix);
+    // handle direct keys
+    while (iterator.hasNext()) {
+      Table.KeyValue<String, OmKeyInfo> kv = iterator.next();
+      String dbKey = kv.getKey();
+      // since the RocksDB is ordered, seek until the prefix isn't matched
+      if (!dbKey.startsWith(seekPrefix)) {
+        break;
+      }
+
+      String[] keys = dbKey.split(OM_KEY_PREFIX);
+
+      // iteration moved to the next level
+      // and not handling direct keys
+      if (keys.length - seekKeys.length > 1) {
+        continue;
+      }
+
+      OmKeyInfo keyInfo = kv.getValue();
+      if (keyInfo != null) {
+        // skip directory markers, just include directKeys
+        if (keyInfo.getKeyName().endsWith(OM_KEY_PREFIX)) {
+          continue;
+        }
+        totalDU += getKeySizeWithReplication(keyInfo);
+      }
+    }
+
+    // handle nested keys (DFS)
+    Set<Long> subDirIds = nsSummary.getChildDir();
+    for (long subDirId: subDirIds) {
+      totalDU += calculateDUUnderObject(subDirId);
+    }
+    return totalDU;
+  }
+
+  /**
+   * This method handles disk usage of direct keys.
+   * @param parentId parent directory/bucket
+   * @param withReplica if withReplica is enabled, set sizeWithReplica
+   * for each direct key's DU
+   * @param listFile if listFile is enabled, append key DU as a subpath
+   * @param duData the current DU data
+   * @param normalizedPath the normalized path request
+   * @return the total DU of all direct keys
+   * @throws IOException IOE
+   */
+  @Override
+  public long handleDirectKeys(long parentId, boolean withReplica,
+                               boolean listFile,
+                               List<DUResponse.DiskUsage> duData,
+                               String normalizedPath) throws IOException {
+
+    Table<String, OmKeyInfo> keyTable = getKeyTable();
+    long keyDataSizeWithReplica = 0L;
+
+    TableIterator<String, ? extends Table.KeyValue<String, OmKeyInfo>>
+        iterator = keyTable.iterator();
+
+    String seekPrefix = OM_KEY_PREFIX +
+        vol +
+        OM_KEY_PREFIX +
+        bucket +
+        OM_KEY_PREFIX;
+
+    NSSummary nsSummary = getReconNamespaceSummaryManager()
+        .getNSSummary(parentId);
+    // empty bucket
+    if (nsSummary == null) {
+      return 0;
+    }
+
+    if (omBucketInfo.getObjectID() != parentId) {
+      String dirName = nsSummary.getDirName();
+      seekPrefix += dirName;
+    }
+    String[] seekKeys = seekPrefix.split(OM_KEY_PREFIX);
+    iterator.seek(seekPrefix);
+
+    while (iterator.hasNext()) {
+      Table.KeyValue<String, OmKeyInfo> kv = iterator.next();
+      String dbKey = kv.getKey();
+
+      if (!dbKey.startsWith(seekPrefix)) {
+        break;
+      }
+
+      String[] keys = dbKey.split(OM_KEY_PREFIX);
+
+      // iteration moved to the next level
+      // and not handling direct keys
+      if (keys.length - seekKeys.length > 1) {
+        continue;
+      }
+
+      OmKeyInfo keyInfo = kv.getValue();
+      if (keyInfo != null) {
+        // skip directory markers, just include directKeys
+        if (keyInfo.getKeyName().endsWith(OM_KEY_PREFIX)) {
+          continue;
+        }
+        DUResponse.DiskUsage diskUsage = new DUResponse.DiskUsage();
+        String subpath = buildSubpath(normalizedPath,
+            keyInfo.getFileName());
+        diskUsage.setSubpath(subpath);
+        diskUsage.setKey(true);
+        diskUsage.setSize(keyInfo.getDataSize());
+
+        if (withReplica) {
+          long keyDU = getKeySizeWithReplication(keyInfo);
+          keyDataSizeWithReplica += keyDU;
+          diskUsage.setSizeWithReplica(keyDU);
+        }
+        // list the key as a subpath
+        if (listFile) {
+          duData.add(diskUsage);
+        }
+      }
+    }
+
+    return keyDataSizeWithReplica;
+  }
+
+  /**
+   * Given a valid path request for a directory,
+   * return the directory object ID.
+   * @param names parsed path request in a list of names
+   * @return directory object ID
+   */
+  @Override
+  public long getDirObjectId(String[] names) throws IOException {
+    return getDirObjectId(names, names.length);
+  }
+
+  /**
+   * Given a valid path request and a cutoff length where should be iterated
+   * up to.
+   * return the directory object ID for the object at the cutoff length
+   * @param names parsed path request in a list of names
+   * @param cutoff cannot be larger than the names' length. If equals,
+   *               return the directory object id for the whole path
+   * @return directory object ID
+   */
+  @Override
+  public long getDirObjectId(String[] names, int cutoff) throws IOException {
+    long dirObjectId = getBucketObjectId(names);
+    StringBuilder bld = new StringBuilder();
+    for (int i = 0; i < cutoff; ++i) {
+      bld.append(OM_KEY_PREFIX)
+          .append(names[i]);
+    }
+    bld.append(OM_KEY_PREFIX);
+    String dirKey = bld.toString();
+    OmKeyInfo dirInfo = getKeyTable().getSkipCache(dirKey);
+
+    try {
+      dirObjectId = dirInfo.getObjectID();
+    } catch (NullPointerException e) {

Review Comment:
   why is the exception being swallowed?  it will cause the bucketid to be returned.



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