|
54 | 54 | import com.amazonaws.SdkBaseException; |
55 | 55 | import com.amazonaws.services.s3.AmazonS3; |
56 | 56 | import com.amazonaws.services.s3.Headers; |
57 | | -import com.amazonaws.services.s3.model.DeleteObjectsRequest; |
58 | | -import com.amazonaws.services.s3.model.DeleteObjectsResult; |
59 | | -import com.amazonaws.services.s3.model.MultiObjectDeleteException; |
60 | 57 | import com.amazonaws.services.s3.model.SelectObjectContentRequest; |
61 | 58 | import com.amazonaws.services.s3.model.SelectObjectContentResult; |
62 | 59 | import com.amazonaws.services.s3.model.UploadPartRequest; |
|
82 | 79 | import software.amazon.awssdk.services.s3.model.ListObjectsV2Request; |
83 | 80 | import software.amazon.awssdk.awscore.exception.AwsServiceException; |
84 | 81 | import software.amazon.awssdk.core.sync.RequestBody; |
| 82 | +import software.amazon.awssdk.services.s3.S3Client; |
85 | 83 | import software.amazon.awssdk.services.s3.model.CopyObjectRequest; |
86 | 84 | import software.amazon.awssdk.services.s3.model.CopyObjectResponse; |
| 85 | +import software.amazon.awssdk.services.s3.model.DeleteObjectsRequest; |
| 86 | +import software.amazon.awssdk.services.s3.model.DeleteObjectsResponse; |
| 87 | +import software.amazon.awssdk.services.s3.model.GetObjectRequest; |
| 88 | +import software.amazon.awssdk.services.s3.model.GetObjectResponse; |
87 | 89 | import software.amazon.awssdk.services.s3.model.HeadObjectRequest; |
88 | 90 | import software.amazon.awssdk.services.s3.model.HeadObjectResponse; |
| 91 | +import software.amazon.awssdk.services.s3.model.ListObjectsRequest; |
| 92 | +import software.amazon.awssdk.services.s3.model.ListObjectsV2Request; |
89 | 93 | import software.amazon.awssdk.services.s3.model.ObjectCannedACL; |
| 94 | +import software.amazon.awssdk.services.s3.model.ObjectIdentifier; |
90 | 95 | import software.amazon.awssdk.services.s3.model.PutObjectRequest; |
91 | 96 | import software.amazon.awssdk.services.s3.model.PutObjectResponse; |
| 97 | +import software.amazon.awssdk.services.s3.model.S3Error; |
92 | 98 | import software.amazon.awssdk.services.s3.model.StorageClass; |
93 | 99 | import software.amazon.awssdk.transfer.s3.CompletedCopy; |
94 | 100 | import software.amazon.awssdk.transfer.s3.CompletedFileUpload; |
@@ -1297,10 +1303,13 @@ public S3Client getAmazonS3V2ClientForTesting(String reason) { |
1297 | 1303 | * Set the client -used in mocking tests to force in a different client. |
1298 | 1304 | * @param client client. |
1299 | 1305 | */ |
1300 | | - protected void setAmazonS3Client(AmazonS3 client) { |
1301 | | - Preconditions.checkNotNull(client, "client"); |
1302 | | - LOG.debug("Setting S3 client to {}", client); |
1303 | | - s3 = client; |
| 1306 | + protected void setAmazonS3Client(Pair<AmazonS3, S3Client> client) { |
| 1307 | + Preconditions.checkNotNull(client.getLeft(), "client"); |
| 1308 | + Preconditions.checkNotNull(client.getRight(), "clientV2"); |
| 1309 | + LOG.debug("Setting S3 client to {}", client.getLeft()); |
| 1310 | + LOG.debug("Setting S3V2 client to {}", client.getRight()); |
| 1311 | + s3 = client.getLeft(); |
| 1312 | + s3V2 = client.getRight(); |
1304 | 1313 |
|
1305 | 1314 | // Need to use a new TransferManager that uses the new client. |
1306 | 1315 | // Also, using a new TransferManager requires a new threadpool as the old |
@@ -2290,7 +2299,7 @@ public CopyObjectResponse copyFile(final String srcKey, |
2290 | 2299 |
|
2291 | 2300 | @Override |
2292 | 2301 | public void removeKeys( |
2293 | | - final List<DeleteObjectsRequest.KeyVersion> keysToDelete, |
| 2302 | + final List<ObjectIdentifier> keysToDelete, |
2294 | 2303 | final boolean deleteFakeDir) |
2295 | 2304 | throws MultiObjectDeleteException, AmazonClientException, IOException { |
2296 | 2305 | auditSpan.activate(); |
@@ -2654,7 +2663,7 @@ protected HeadObjectResponse getObjectMetadata(String key, |
2654 | 2663 | changeTracker.processMetadata(headObjectResponse, operation); |
2655 | 2664 | } |
2656 | 2665 | return headObjectResponse; |
2657 | | - } catch(AwsServiceException ase) { |
| 2666 | + } catch (AwsServiceException ase) { |
2658 | 2667 | if (!isObjectNotFound(ase)) { |
2659 | 2668 | // file not found is not considered a failure of the call, |
2660 | 2669 | // so only switch the duration tracker to update failure |
@@ -2807,8 +2816,9 @@ protected void deleteObject(String key) |
2807 | 2816 | incrementStatistic(OBJECT_DELETE_OBJECTS); |
2808 | 2817 | trackDurationOfInvocation(getDurationTrackerFactory(), |
2809 | 2818 | OBJECT_DELETE_REQUEST.getSymbol(), |
2810 | | - () -> s3.deleteObject(getRequestFactory() |
2811 | | - .newDeleteObjectRequest(key))); |
| 2819 | + () -> s3V2.deleteObject(getRequestFactory() |
| 2820 | + .newDeleteObjectRequestBuilder(key) |
| 2821 | + .build())); |
2812 | 2822 | return null; |
2813 | 2823 | }); |
2814 | 2824 | } |
@@ -2870,40 +2880,43 @@ private void blockRootDelete(String key) throws InvalidRequestException { |
2870 | 2880 | * @return the AWS response |
2871 | 2881 | * @throws MultiObjectDeleteException one or more of the keys could not |
2872 | 2882 | * be deleted. |
2873 | | - * @throws AmazonClientException amazon-layer failure. |
| 2883 | + * @throws AwsServiceException amazon-layer failure. |
2874 | 2884 | */ |
2875 | 2885 | @Retries.RetryRaw |
2876 | | - private DeleteObjectsResult deleteObjects(DeleteObjectsRequest deleteRequest) |
2877 | | - throws MultiObjectDeleteException, AmazonClientException, IOException { |
| 2886 | + private DeleteObjectsResponse deleteObjects(DeleteObjectsRequest deleteRequest) |
| 2887 | + throws MultiObjectDeleteException, AwsServiceException, IOException { |
2878 | 2888 | incrementWriteOperations(); |
2879 | 2889 | BulkDeleteRetryHandler retryHandler = |
2880 | 2890 | new BulkDeleteRetryHandler(createStoreContext()); |
2881 | | - int keyCount = deleteRequest.getKeys().size(); |
2882 | | - try(DurationInfo ignored = |
| 2891 | + int keyCount = deleteRequest.delete().objects().size(); |
| 2892 | + try (DurationInfo ignored = |
2883 | 2893 | new DurationInfo(LOG, false, "DELETE %d keys", |
2884 | 2894 | keyCount)) { |
2885 | | - return invoker.retryUntranslated("delete", |
2886 | | - DELETE_CONSIDERED_IDEMPOTENT, |
2887 | | - (text, e, r, i) -> { |
2888 | | - // handle the failure |
2889 | | - retryHandler.bulkDeleteRetried(deleteRequest, e); |
2890 | | - }, |
2891 | | - // duration is tracked in the bulk delete counters |
2892 | | - trackDurationOfOperation(getDurationTrackerFactory(), |
2893 | | - OBJECT_BULK_DELETE_REQUEST.getSymbol(), () -> { |
| 2895 | + DeleteObjectsResponse response = |
| 2896 | + invoker.retryUntranslated("delete", DELETE_CONSIDERED_IDEMPOTENT, |
| 2897 | + (text, e, r, i) -> { |
| 2898 | + // handle the failure |
| 2899 | + retryHandler.bulkDeleteRetried(deleteRequest, e); |
| 2900 | + }, |
| 2901 | + // duration is tracked in the bulk delete counters |
| 2902 | + trackDurationOfOperation(getDurationTrackerFactory(), |
| 2903 | + OBJECT_BULK_DELETE_REQUEST.getSymbol(), () -> { |
2894 | 2904 | incrementStatistic(OBJECT_DELETE_OBJECTS, keyCount); |
2895 | | - return s3.deleteObjects(deleteRequest); |
2896 | | - })); |
2897 | | - } catch (MultiObjectDeleteException e) { |
2898 | | - // one or more of the keys could not be deleted. |
2899 | | - // log and rethrow |
2900 | | - List<MultiObjectDeleteException.DeleteError> errors = e.getErrors(); |
2901 | | - LOG.debug("Partial failure of delete, {} errors", errors.size(), e); |
2902 | | - for (MultiObjectDeleteException.DeleteError error : errors) { |
2903 | | - LOG.debug("{}: \"{}\" - {}", |
2904 | | - error.getKey(), error.getCode(), error.getMessage()); |
| 2905 | + return s3V2.deleteObjects(deleteRequest); |
| 2906 | + })); |
| 2907 | + |
| 2908 | + if (!response.errors().isEmpty()) { |
| 2909 | + // one or more of the keys could not be deleted. |
| 2910 | + // log and then throw |
| 2911 | + List<S3Error> errors = response.errors(); |
| 2912 | + LOG.debug("Partial failure of delete, {} errors", errors.size()); |
| 2913 | + for (S3Error error : errors) { |
| 2914 | + LOG.debug("{}: \"{}\" - {}", error.key(), error.code(), error.message()); |
| 2915 | + } |
| 2916 | + throw new MultiObjectDeleteException(errors); |
2905 | 2917 | } |
2906 | | - throw e; |
| 2918 | + |
| 2919 | + return response; |
2907 | 2920 | } |
2908 | 2921 | } |
2909 | 2922 |
|
@@ -3104,56 +3117,57 @@ public void incrementPutProgressStatistics(String key, long bytes) { |
3104 | 3117 | * be deleted in a multiple object delete operation. |
3105 | 3118 | * The number of rejected objects will be added to the metric |
3106 | 3119 | * {@link Statistic#FILES_DELETE_REJECTED}. |
3107 | | - * @throws AmazonClientException other amazon-layer failure. |
| 3120 | + * @throws AwsServiceException other amazon-layer failure. |
3108 | 3121 | */ |
3109 | 3122 | @Retries.RetryRaw |
3110 | 3123 | private void removeKeysS3( |
3111 | | - List<DeleteObjectsRequest.KeyVersion> keysToDelete, |
| 3124 | + List<ObjectIdentifier> keysToDelete, |
3112 | 3125 | boolean deleteFakeDir) |
3113 | | - throws MultiObjectDeleteException, AmazonClientException, |
3114 | | - IOException { |
| 3126 | + throws MultiObjectDeleteException, AwsServiceException, IOException { |
3115 | 3127 | if (LOG.isDebugEnabled()) { |
3116 | 3128 | LOG.debug("Initiating delete operation for {} objects", |
3117 | 3129 | keysToDelete.size()); |
3118 | | - for (DeleteObjectsRequest.KeyVersion key : keysToDelete) { |
3119 | | - LOG.debug(" {} {}", key.getKey(), |
3120 | | - key.getVersion() != null ? key.getVersion() : ""); |
| 3130 | + for (ObjectIdentifier objectIdentifier : keysToDelete) { |
| 3131 | + LOG.debug(" {} {}", objectIdentifier.key(), |
| 3132 | + objectIdentifier.versionId() != null ? objectIdentifier.versionId() : ""); |
3121 | 3133 | } |
3122 | 3134 | } |
3123 | 3135 | if (keysToDelete.isEmpty()) { |
3124 | 3136 | // exit fast if there are no keys to delete |
3125 | 3137 | return; |
3126 | 3138 | } |
3127 | | - for (DeleteObjectsRequest.KeyVersion keyVersion : keysToDelete) { |
3128 | | - blockRootDelete(keyVersion.getKey()); |
| 3139 | + for (ObjectIdentifier objectIdentifier : keysToDelete) { |
| 3140 | + blockRootDelete(objectIdentifier.key()); |
3129 | 3141 | } |
3130 | 3142 | try { |
3131 | 3143 | if (enableMultiObjectsDelete) { |
3132 | 3144 | if (keysToDelete.size() <= pageSize) { |
3133 | 3145 | deleteObjects(getRequestFactory() |
3134 | | - .newBulkDeleteRequest(keysToDelete)); |
| 3146 | + .newBulkDeleteRequestBuilder(keysToDelete) |
| 3147 | + .build()); |
3135 | 3148 | } else { |
3136 | 3149 | // Multi object deletion of more than 1000 keys is not supported |
3137 | 3150 | // by s3. So we are paging the keys by page size. |
3138 | 3151 | LOG.debug("Partitioning the keys to delete as it is more than " + |
3139 | 3152 | "page size. Number of keys: {}, Page size: {}", |
3140 | 3153 | keysToDelete.size(), pageSize); |
3141 | | - for (List<DeleteObjectsRequest.KeyVersion> batchOfKeysToDelete : |
| 3154 | + for (List<ObjectIdentifier> batchOfKeysToDelete : |
3142 | 3155 | Lists.partition(keysToDelete, pageSize)) { |
3143 | 3156 | deleteObjects(getRequestFactory() |
3144 | | - .newBulkDeleteRequest(batchOfKeysToDelete)); |
| 3157 | + .newBulkDeleteRequestBuilder(batchOfKeysToDelete) |
| 3158 | + .build()); |
3145 | 3159 | } |
3146 | 3160 | } |
3147 | 3161 | } else { |
3148 | | - for (DeleteObjectsRequest.KeyVersion keyVersion : keysToDelete) { |
3149 | | - deleteObject(keyVersion.getKey()); |
| 3162 | + for (ObjectIdentifier objectIdentifier : keysToDelete) { |
| 3163 | + deleteObject(objectIdentifier.key()); |
3150 | 3164 | } |
3151 | 3165 | } |
3152 | 3166 | } catch (MultiObjectDeleteException ex) { |
3153 | 3167 | // partial delete. |
3154 | 3168 | // Update the stats with the count of the actual number of successful |
3155 | 3169 | // deletions. |
3156 | | - int rejected = ex.getErrors().size(); |
| 3170 | + int rejected = ex.errors().size(); |
3157 | 3171 | noteDeleted(keysToDelete.size() - rejected, deleteFakeDir); |
3158 | 3172 | incrementStatistic(FILES_DELETE_REJECTED, rejected); |
3159 | 3173 | throw ex; |
@@ -3186,15 +3200,15 @@ private void noteDeleted(final int count, final boolean deleteFakeDir) { |
3186 | 3200 | * a mistaken attempt to delete the root directory. |
3187 | 3201 | * @throws MultiObjectDeleteException one or more of the keys could not |
3188 | 3202 | * be deleted in a multiple object delete operation. |
3189 | | - * @throws AmazonClientException amazon-layer failure. |
| 3203 | + * @throws AwsServiceException amazon-layer failure. |
3190 | 3204 | * @throws IOException other IO Exception. |
3191 | 3205 | */ |
3192 | 3206 | @VisibleForTesting |
3193 | 3207 | @Retries.RetryRaw |
3194 | 3208 | public void removeKeys( |
3195 | | - final List<DeleteObjectsRequest.KeyVersion> keysToDelete, |
| 3209 | + final List<ObjectIdentifier> keysToDelete, |
3196 | 3210 | final boolean deleteFakeDir) |
3197 | | - throws MultiObjectDeleteException, AmazonClientException, |
| 3211 | + throws MultiObjectDeleteException, AwsServiceException, |
3198 | 3212 | IOException { |
3199 | 3213 | try (DurationInfo ignored = new DurationInfo(LOG, false, |
3200 | 3214 | "Deleting %d keys", keysToDelete.size())) { |
@@ -4418,22 +4432,22 @@ private PutObjectOptions putOptionsForPath(Path path) { |
4418 | 4432 | */ |
4419 | 4433 | @Retries.RetryExceptionsSwallowed |
4420 | 4434 | private void deleteUnnecessaryFakeDirectories(Path path) { |
4421 | | - List<DeleteObjectsRequest.KeyVersion> keysToRemove = new ArrayList<>(); |
| 4435 | + List<ObjectIdentifier> keysToRemove = new ArrayList<>(); |
4422 | 4436 | while (!path.isRoot()) { |
4423 | 4437 | String key = pathToKey(path); |
4424 | 4438 | key = (key.endsWith("/")) ? key : (key + "/"); |
4425 | 4439 | LOG.trace("To delete unnecessary fake directory {} for {}", key, path); |
4426 | | - keysToRemove.add(new DeleteObjectsRequest.KeyVersion(key)); |
| 4440 | + keysToRemove.add(ObjectIdentifier.builder().key(key).build()); |
4427 | 4441 | path = path.getParent(); |
4428 | 4442 | } |
4429 | 4443 | try { |
4430 | 4444 | removeKeys(keysToRemove, true); |
4431 | | - } catch(AmazonClientException | IOException e) { |
| 4445 | + } catch (AwsServiceException | IOException e) { |
4432 | 4446 | instrumentation.errorIgnored(); |
4433 | 4447 | if (LOG.isDebugEnabled()) { |
4434 | 4448 | StringBuilder sb = new StringBuilder(); |
4435 | | - for(DeleteObjectsRequest.KeyVersion kv : keysToRemove) { |
4436 | | - sb.append(kv.getKey()).append(","); |
| 4449 | + for (ObjectIdentifier objectIdentifier : keysToRemove) { |
| 4450 | + sb.append(objectIdentifier.key()).append(","); |
4437 | 4451 | } |
4438 | 4452 | LOG.debug("While deleting keys {} ", sb.toString(), e); |
4439 | 4453 | } |
|
0 commit comments