xichen01 commented on code in PR #8974:
URL: https://github.com/apache/ozone/pull/8974#discussion_r2328761632


##########
hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/service/KeyLifecycleService.java:
##########
@@ -295,8 +306,8 @@ public BackgroundTaskResult call() {
               expiredKeyList, false);
         } else if (ozoneTrash != null) {
           // move keys to trash
-          // TODO: add unit test in next patch
-          moveKeysToTrash(bucket.getVolumeName(), bucket.getBucketName(), 
expiredKeyList);
+          // TODO: move directory to trash in next patch
+          moveKeysToTrash(bucket, expiredKeyList);

Review Comment:
   The if-else branch in L272 only deletes keys via 
`sendDeleteKeysRequestAndClearList` and without the “move to trash” logic. 
   Therefore, here can only move the remaining undeleted keys from L272 to the 
trash.
   Does this seem problematic?



##########
hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/service/KeyLifecycleService.java:
##########
@@ -628,45 +674,178 @@ private void sendDeleteKeysRequestAndClearList(String 
volume, String bucket,
                 getOzoneManager(), omRequest, clientId, 
callId.getAndIncrement());
             long endTime = System.nanoTime();
             LOG.debug("DeleteKeys request with {} keys cost {} ns", keyCount, 
endTime - startTime);
-            i += batchSize;
-            startIndex += batchSize;
+            long deletedCount = keyCount;
+            long deletedSize = keysList.replicatedSizeSubList(startIndex, 
endIndex)
+                .stream().mapToLong(Long::longValue).sum();
             if (response != null) {
               if (!response.getSuccess()) {
                 // log the failure and continue the iterating
-                LOG.error("DeleteKeys request failed with " + 
response.getMessage() +
-                    " for volume: {}, bucket: {}", volume, bucket);
-                continue;
+                LOG.error("DeleteKeys request " + response.getStatus() + " 
failed for volume: {}, bucket: {}",
+                    volume, bucket);
+                if (response.getDeleteKeysResponse().hasUnDeletedKeys()) {
+                  DeleteKeyArgs unDeletedKeys = 
response.getDeleteKeysResponse().getUnDeletedKeys();
+                  for (String key : unDeletedKeys.getKeysList()) {
+                    Long size = keysList.getReplicatedSize(key);
+                    if (size == null) {
+                      LOG.error("Undeleted key {}/{}/{} doesn't in keyLists", 
volume, bucket, key);
+                      continue;
+                    }
+                    deletedCount -= 1;
+                    deletedSize -= size;
+                  }
+                }
+                for (DeleteKeyError e : 
response.getDeleteKeysResponse().getErrorsList()) {
+                  Long size = keysList.getReplicatedSize(e.getKey());
+                  if (size == null) {
+                    LOG.error("Deleted error key {}/{}/{} doesn't in 
keyLists", volume, bucket, e.getKey());
+                    continue;
+                  }
+                  deletedCount -= 1;
+                  deletedSize -= size;
+                }
               } else {
                 LOG.debug("DeleteKeys request of total {} keys, {} not 
deleted", keyCount,
                     response.getDeleteKeysResponse().getErrorsCount());
               }
             }
             if (dir) {
-              numDirDeleted += keyCount;
-              metrics.incrNumDirDeleted(keyCount);
+              numDirDeleted += deletedCount;
+              metrics.incrNumDirDeleted(deletedCount);
             } else {
-              numKeyDeleted += keyCount;
-              metrics.incrNumKeyDeleted(keyCount);
+              numKeyDeleted += deletedCount;
+              sizeKeyDeleted += deletedSize;
+              metrics.incrNumKeyDeleted(deletedCount);
+              metrics.incrSizeKeyDeleted(deletedSize);
             }
+            i += keyCount;
+            startIndex += keyCount;
           } else {
             batchSize /= 2;
           }
         }
-        keysList.clear();
       } catch (ServiceException e) {
         LOG.error("Failed to send DeleteKeysRequest", e);
+      } finally {
+        keysList.clear();
       }
     }
 
-    private void moveKeysToTrash(String volume, String bucket, 
LimitedExpiredObjectList keysList) {
-      for (int index = 0; index < keysList.size(); index++) {
+    private void moveKeysToTrash(OmBucketInfo bucket, LimitedExpiredObjectList 
keysList) {
+      if (keysList.isEmpty()) {
+        return;
+      }
+      String volumeName = bucket.getVolumeName();
+      String bucketName = bucket.getBucketName();
+      String trashCurrent;
+      try {
+        trashCurrent = checkAndCreateTrashDirectoryIfNeeded(bucket);
+      } catch (IOException e) {
+        keysList.clear();
+        return;
+      }
+
+      for (int i = 0; i < keysList.size(); i++) {
+        String keyName = keysList.getName(i);
+        String targetKeyName = trashCurrent + OM_KEY_PREFIX + keyName;
+        KeyArgs keyArgs = KeyArgs.newBuilder().setKeyName(keyName)
+            .setVolumeName(volumeName).setBucketName(bucketName).build();
+
+        /**
+         * Trash examples:
+         * /s3v/test/readme ->  /s3v/test/.Trash/hadoop/Current/readme
+         * /s3v/test/dir1/readme -> /s3v/test/.Trash/hadoop/Current/dir1/readme
+         */
+        RenameKeyRequest renameKeyRequest = RenameKeyRequest.newBuilder()
+            .setKeyArgs(keyArgs)
+            .setToKeyName(targetKeyName)
+            .setUpdateID(keysList.getUpdateID(i))
+            .build();
+
+        // send request out
+        OMRequest omRequest = OMRequest.newBuilder()
+            .setCmdType(OzoneManagerProtocolProtos.Type.RenameKey)
+            .setVersion(ClientVersion.CURRENT_VERSION)
+            .setClientId(clientId.toString())
+            .setRenameKeyRequest(renameKeyRequest)
+            .build();
         try {
-          ozoneTrash.moveToTrash(new Path(OM_KEY_PREFIX + volume + 
OM_KEY_PREFIX + bucket + OM_KEY_PREFIX +
-              keysList.getName(index)));
-        } catch (IOException e) {
-          // log failure and continue
-          LOG.warn("Failed to move key {} to trash", keysList.getName(index), 
e);
+          // perform preExecute as ratis submit do no perform preExecute
+          OMClientRequest omClientRequest = 
OzoneManagerRatisUtils.createClientRequest(omRequest, ozoneManager);
+          omRequest = omClientRequest.preExecute(ozoneManager);
+          final OzoneManagerProtocolProtos.OMResponse response = 
OzoneManagerRatisUtils.submitRequest(
+              getOzoneManager(), omRequest, clientId, 
callId.getAndIncrement());
+          if (response != null) {
+            if (!response.getSuccess()) {
+              // log the failure and continue the iterating
+              LOG.error("RenameKey request failed with source key: {}, dest 
key: {}", keyName, targetKeyName);
+              continue;
+            }
+          }
+          LOG.info("RenameKey request succeed with source key: {}, dest key: 
{}", keyName, targetKeyName);

Review Comment:
   This will generate many logs, we can set it to debug level.



##########
hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/service/KeyLifecycleService.java:
##########
@@ -628,45 +674,178 @@ private void sendDeleteKeysRequestAndClearList(String 
volume, String bucket,
                 getOzoneManager(), omRequest, clientId, 
callId.getAndIncrement());
             long endTime = System.nanoTime();
             LOG.debug("DeleteKeys request with {} keys cost {} ns", keyCount, 
endTime - startTime);
-            i += batchSize;
-            startIndex += batchSize;
+            long deletedCount = keyCount;
+            long deletedSize = keysList.replicatedSizeSubList(startIndex, 
endIndex)
+                .stream().mapToLong(Long::longValue).sum();
             if (response != null) {
               if (!response.getSuccess()) {
                 // log the failure and continue the iterating
-                LOG.error("DeleteKeys request failed with " + 
response.getMessage() +
-                    " for volume: {}, bucket: {}", volume, bucket);
-                continue;
+                LOG.error("DeleteKeys request " + response.getStatus() + " 
failed for volume: {}, bucket: {}",
+                    volume, bucket);
+                if (response.getDeleteKeysResponse().hasUnDeletedKeys()) {
+                  DeleteKeyArgs unDeletedKeys = 
response.getDeleteKeysResponse().getUnDeletedKeys();
+                  for (String key : unDeletedKeys.getKeysList()) {
+                    Long size = keysList.getReplicatedSize(key);
+                    if (size == null) {
+                      LOG.error("Undeleted key {}/{}/{} doesn't in keyLists", 
volume, bucket, key);
+                      continue;
+                    }
+                    deletedCount -= 1;
+                    deletedSize -= size;
+                  }
+                }
+                for (DeleteKeyError e : 
response.getDeleteKeysResponse().getErrorsList()) {
+                  Long size = keysList.getReplicatedSize(e.getKey());
+                  if (size == null) {
+                    LOG.error("Deleted error key {}/{}/{} doesn't in 
keyLists", volume, bucket, e.getKey());
+                    continue;
+                  }
+                  deletedCount -= 1;
+                  deletedSize -= size;
+                }
               } else {
                 LOG.debug("DeleteKeys request of total {} keys, {} not 
deleted", keyCount,
                     response.getDeleteKeysResponse().getErrorsCount());
               }
             }
             if (dir) {
-              numDirDeleted += keyCount;
-              metrics.incrNumDirDeleted(keyCount);
+              numDirDeleted += deletedCount;
+              metrics.incrNumDirDeleted(deletedCount);
             } else {
-              numKeyDeleted += keyCount;
-              metrics.incrNumKeyDeleted(keyCount);
+              numKeyDeleted += deletedCount;
+              sizeKeyDeleted += deletedSize;
+              metrics.incrNumKeyDeleted(deletedCount);
+              metrics.incrSizeKeyDeleted(deletedSize);
             }
+            i += keyCount;
+            startIndex += keyCount;
           } else {
             batchSize /= 2;
           }
         }
-        keysList.clear();
       } catch (ServiceException e) {
         LOG.error("Failed to send DeleteKeysRequest", e);
+      } finally {
+        keysList.clear();
       }
     }
 
-    private void moveKeysToTrash(String volume, String bucket, 
LimitedExpiredObjectList keysList) {
-      for (int index = 0; index < keysList.size(); index++) {
+    private void moveKeysToTrash(OmBucketInfo bucket, LimitedExpiredObjectList 
keysList) {
+      if (keysList.isEmpty()) {
+        return;
+      }
+      String volumeName = bucket.getVolumeName();
+      String bucketName = bucket.getBucketName();
+      String trashCurrent;
+      try {
+        trashCurrent = checkAndCreateTrashDirectoryIfNeeded(bucket);
+      } catch (IOException e) {
+        keysList.clear();
+        return;
+      }
+
+      for (int i = 0; i < keysList.size(); i++) {
+        String keyName = keysList.getName(i);
+        String targetKeyName = trashCurrent + OM_KEY_PREFIX + keyName;
+        KeyArgs keyArgs = KeyArgs.newBuilder().setKeyName(keyName)
+            .setVolumeName(volumeName).setBucketName(bucketName).build();
+
+        /**
+         * Trash examples:
+         * /s3v/test/readme ->  /s3v/test/.Trash/hadoop/Current/readme
+         * /s3v/test/dir1/readme -> /s3v/test/.Trash/hadoop/Current/dir1/readme
+         */
+        RenameKeyRequest renameKeyRequest = RenameKeyRequest.newBuilder()
+            .setKeyArgs(keyArgs)
+            .setToKeyName(targetKeyName)
+            .setUpdateID(keysList.getUpdateID(i))
+            .build();
+
+        // send request out
+        OMRequest omRequest = OMRequest.newBuilder()
+            .setCmdType(OzoneManagerProtocolProtos.Type.RenameKey)
+            .setVersion(ClientVersion.CURRENT_VERSION)
+            .setClientId(clientId.toString())
+            .setRenameKeyRequest(renameKeyRequest)
+            .build();
         try {
-          ozoneTrash.moveToTrash(new Path(OM_KEY_PREFIX + volume + 
OM_KEY_PREFIX + bucket + OM_KEY_PREFIX +
-              keysList.getName(index)));
-        } catch (IOException e) {
-          // log failure and continue
-          LOG.warn("Failed to move key {} to trash", keysList.getName(index), 
e);
+          // perform preExecute as ratis submit do no perform preExecute
+          OMClientRequest omClientRequest = 
OzoneManagerRatisUtils.createClientRequest(omRequest, ozoneManager);
+          omRequest = omClientRequest.preExecute(ozoneManager);

Review Comment:
   should we `doAs` too, just like the `checkAndCreateTrashDirectoryIfNeeded`



##########
hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/service/KeyLifecycleService.java:
##########
@@ -628,45 +674,178 @@ private void sendDeleteKeysRequestAndClearList(String 
volume, String bucket,
                 getOzoneManager(), omRequest, clientId, 
callId.getAndIncrement());
             long endTime = System.nanoTime();
             LOG.debug("DeleteKeys request with {} keys cost {} ns", keyCount, 
endTime - startTime);
-            i += batchSize;
-            startIndex += batchSize;
+            long deletedCount = keyCount;
+            long deletedSize = keysList.replicatedSizeSubList(startIndex, 
endIndex)
+                .stream().mapToLong(Long::longValue).sum();
             if (response != null) {
               if (!response.getSuccess()) {
                 // log the failure and continue the iterating
-                LOG.error("DeleteKeys request failed with " + 
response.getMessage() +
-                    " for volume: {}, bucket: {}", volume, bucket);
-                continue;
+                LOG.error("DeleteKeys request " + response.getStatus() + " 
failed for volume: {}, bucket: {}",
+                    volume, bucket);
+                if (response.getDeleteKeysResponse().hasUnDeletedKeys()) {
+                  DeleteKeyArgs unDeletedKeys = 
response.getDeleteKeysResponse().getUnDeletedKeys();
+                  for (String key : unDeletedKeys.getKeysList()) {
+                    Long size = keysList.getReplicatedSize(key);
+                    if (size == null) {
+                      LOG.error("Undeleted key {}/{}/{} doesn't in keyLists", 
volume, bucket, key);
+                      continue;
+                    }
+                    deletedCount -= 1;
+                    deletedSize -= size;
+                  }
+                }
+                for (DeleteKeyError e : 
response.getDeleteKeysResponse().getErrorsList()) {
+                  Long size = keysList.getReplicatedSize(e.getKey());
+                  if (size == null) {
+                    LOG.error("Deleted error key {}/{}/{} doesn't in 
keyLists", volume, bucket, e.getKey());
+                    continue;
+                  }
+                  deletedCount -= 1;
+                  deletedSize -= size;
+                }
               } else {
                 LOG.debug("DeleteKeys request of total {} keys, {} not 
deleted", keyCount,
                     response.getDeleteKeysResponse().getErrorsCount());
               }
             }
             if (dir) {
-              numDirDeleted += keyCount;
-              metrics.incrNumDirDeleted(keyCount);
+              numDirDeleted += deletedCount;
+              metrics.incrNumDirDeleted(deletedCount);
             } else {
-              numKeyDeleted += keyCount;
-              metrics.incrNumKeyDeleted(keyCount);
+              numKeyDeleted += deletedCount;
+              sizeKeyDeleted += deletedSize;
+              metrics.incrNumKeyDeleted(deletedCount);
+              metrics.incrSizeKeyDeleted(deletedSize);
             }
+            i += keyCount;
+            startIndex += keyCount;
           } else {
             batchSize /= 2;
           }
         }
-        keysList.clear();
       } catch (ServiceException e) {
         LOG.error("Failed to send DeleteKeysRequest", e);
+      } finally {
+        keysList.clear();
       }
     }
 
-    private void moveKeysToTrash(String volume, String bucket, 
LimitedExpiredObjectList keysList) {
-      for (int index = 0; index < keysList.size(); index++) {
+    private void moveKeysToTrash(OmBucketInfo bucket, LimitedExpiredObjectList 
keysList) {
+      if (keysList.isEmpty()) {
+        return;
+      }
+      String volumeName = bucket.getVolumeName();
+      String bucketName = bucket.getBucketName();
+      String trashCurrent;
+      try {
+        trashCurrent = checkAndCreateTrashDirectoryIfNeeded(bucket);
+      } catch (IOException e) {
+        keysList.clear();
+        return;
+      }
+
+      for (int i = 0; i < keysList.size(); i++) {
+        String keyName = keysList.getName(i);
+        String targetKeyName = trashCurrent + OM_KEY_PREFIX + keyName;
+        KeyArgs keyArgs = KeyArgs.newBuilder().setKeyName(keyName)
+            .setVolumeName(volumeName).setBucketName(bucketName).build();
+
+        /**
+         * Trash examples:
+         * /s3v/test/readme ->  /s3v/test/.Trash/hadoop/Current/readme
+         * /s3v/test/dir1/readme -> /s3v/test/.Trash/hadoop/Current/dir1/readme
+         */
+        RenameKeyRequest renameKeyRequest = RenameKeyRequest.newBuilder()
+            .setKeyArgs(keyArgs)
+            .setToKeyName(targetKeyName)
+            .setUpdateID(keysList.getUpdateID(i))
+            .build();
+
+        // send request out
+        OMRequest omRequest = OMRequest.newBuilder()
+            .setCmdType(OzoneManagerProtocolProtos.Type.RenameKey)
+            .setVersion(ClientVersion.CURRENT_VERSION)
+            .setClientId(clientId.toString())
+            .setRenameKeyRequest(renameKeyRequest)
+            .build();
         try {
-          ozoneTrash.moveToTrash(new Path(OM_KEY_PREFIX + volume + 
OM_KEY_PREFIX + bucket + OM_KEY_PREFIX +
-              keysList.getName(index)));
-        } catch (IOException e) {
-          // log failure and continue
-          LOG.warn("Failed to move key {} to trash", keysList.getName(index), 
e);
+          // perform preExecute as ratis submit do no perform preExecute
+          OMClientRequest omClientRequest = 
OzoneManagerRatisUtils.createClientRequest(omRequest, ozoneManager);
+          omRequest = omClientRequest.preExecute(ozoneManager);
+          final OzoneManagerProtocolProtos.OMResponse response = 
OzoneManagerRatisUtils.submitRequest(
+              getOzoneManager(), omRequest, clientId, 
callId.getAndIncrement());
+          if (response != null) {
+            if (!response.getSuccess()) {
+              // log the failure and continue the iterating
+              LOG.error("RenameKey request failed with source key: {}, dest 
key: {}", keyName, targetKeyName);
+              continue;
+            }
+          }
+          LOG.info("RenameKey request succeed with source key: {}, dest key: 
{}", keyName, targetKeyName);
+          numKeyRenamed += 1;
+          sizeKeyRenamed += keysList.getReplicatedSize(i);
+          metrics.incrNumKeyRenamed(1);
+          metrics.incrSizeKeyRenamed(keysList.getReplicatedSize(i));
+        } catch (ServiceException | IOException e) {
+          LOG.error("Failed to send RenameKeysRequest", e);
+        }
+      }
+      keysList.clear();
+    }
+
+    private String checkAndCreateTrashDirectoryIfNeeded(OmBucketInfo bucket) 
throws IOException {
+      String userTrashRoot = TRASH_PREFIX + OM_KEY_PREFIX + bucket.getOwner();
+      String userTrashCurrent = userTrashRoot + OM_KEY_PREFIX + CURRENT;
+      try {
+        OmKeyArgs key = new 
OmKeyArgs.Builder().setVolumeName(bucket.getVolumeName())
+            .setBucketName(bucket.getBucketName()).setKeyName(userTrashCurrent)
+            .setOwnerName(bucket.getOwner()).build();
+        ozoneManager.getFileStatus(key);
+        return userTrashCurrent;
+      } catch (IOException e) {
+        if (e instanceof OMException &&
+            (((OMException) e).getResult() == 
OMException.ResultCodes.FILE_NOT_FOUND ||
+                ((OMException) e).getResult() == 
OMException.ResultCodes.DIRECTORY_NOT_FOUND)) {
+          // create the trash/Current directory for user
+          KeyArgs keyArgs = 
KeyArgs.newBuilder().setVolumeName(bucket.getVolumeName())
+              
.setBucketName(bucket.getBucketName()).setKeyName(userTrashCurrent)
+              .setOwnerName(bucket.getOwner()).setRecursive(true).build();
+          OMRequest omRequest = 
OMRequest.newBuilder().setCreateDirectoryRequest(
+                  CreateDirectoryRequest.newBuilder().setKeyArgs(keyArgs))
+              .setCmdType(OzoneManagerProtocolProtos.Type.CreateDirectory)
+              .setVersion(ClientVersion.CURRENT_VERSION)
+              .setClientId(clientId.toString())
+              .build();
+          try {
+            // perform preExecute as ratis submit do no perform preExecute
+            final OMClientRequest omClientRequest = 
OzoneManagerRatisUtils.createClientRequest(omRequest, ozoneManager);
+            UserGroupInformation ugi = 
UserGroupInformation.createRemoteUser(bucket.getOwner());
+            OzoneManagerProtocolProtos.OMResponse omResponse =
+                ugi.doAs(new 
PrivilegedExceptionAction<OzoneManagerProtocolProtos.OMResponse>() {
+                  @Override
+                  public OzoneManagerProtocolProtos.OMResponse run() throws 
Exception {
+                    OMRequest request = 
omClientRequest.preExecute(ozoneManager);
+                    return 
OzoneManagerRatisUtils.submitRequest(getOzoneManager(),
+                        request, clientId, callId.getAndIncrement());
+                  }
+                });
+
+            if (omResponse != null) {
+              if (!omResponse.getSuccess()) {
+                LOG.error("CreateDirectory request failed with {}, path: {}",
+                    omResponse.getMessage(), userTrashCurrent);
+                throw new IOException("Failed to create trash directory " + 
userTrashCurrent);
+              }
+              LOG.error("Create trash current directory: {}", 
userTrashCurrent);

Review Comment:
   Should be info ?



-- 
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: [email protected]

For queries about this service, please contact Infrastructure at:
[email protected]


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

Reply via email to