NIFI-9694 : Correct FetchS3Object.java provenance data URL

Current provenance data URL is incorrect as it's hardcoded.
Follow PutS3Object.java method, using AmazonS3Client (instead of AmazonS3) to use getResourceUrl(bucket,key) to build the correct URL for provenance data

This closes #5779

Signed-off-by: Mike Thomsen <mthomsen@apache.org>
This commit is contained in:
lawye1973 2022-02-17 22:44:19 +08:00 committed by Mike Thomsen
parent dfbf2e3cea
commit 8d80fb2930
No known key found for this signature in database
GPG Key ID: 88511C3D4CAD246F
1 changed files with 5 additions and 4 deletions

View File

@ -17,7 +17,7 @@
package org.apache.nifi.processors.aws.s3;
import com.amazonaws.AmazonClientException;
import com.amazonaws.services.s3.AmazonS3;
import com.amazonaws.services.s3.AmazonS3Client;
import com.amazonaws.services.s3.model.GetObjectMetadataRequest;
import com.amazonaws.services.s3.model.GetObjectRequest;
import com.amazonaws.services.s3.model.ObjectMetadata;
@ -163,7 +163,7 @@ public class FetchS3Object extends AbstractS3Processor {
final String bucket = context.getProperty(BUCKET).evaluateAttributeExpressions(attributes).getValue();
final String key = context.getProperty(KEY).evaluateAttributeExpressions(attributes).getValue();
final AmazonS3 client = getConfiguration(context).getClient();
final AmazonS3Client client = getConfiguration(context).getClient();
final GetObjectMetadataRequest request = createGetObjectMetadataRequest(context, attributes);
try {
@ -204,7 +204,7 @@ public class FetchS3Object extends AbstractS3Processor {
final String bucket = context.getProperty(BUCKET).evaluateAttributeExpressions(flowFile).getValue();
final String key = context.getProperty(KEY).evaluateAttributeExpressions(flowFile).getValue();
final AmazonS3 client = getClient();
final AmazonS3Client client = getClient();
final GetObjectRequest request = createGetObjectRequest(context, flowFile.getAttributes());
try (final S3Object s3Object = client.getObject(request)) {
@ -277,9 +277,10 @@ public class FetchS3Object extends AbstractS3Processor {
}
session.transfer(flowFile, REL_SUCCESS);
final String url = client.getResourceUrl(bucket, key);
final long transferMillis = TimeUnit.NANOSECONDS.toMillis(System.nanoTime() - startNanos);
getLogger().info("Successfully retrieved S3 Object for {} in {} millis; routing to success", new Object[]{flowFile, transferMillis});
session.getProvenanceReporter().fetch(flowFile, "http://" + bucket + ".amazonaws.com/" + key, transferMillis);
session.getProvenanceReporter().fetch(flowFile, url, transferMillis);
}
private GetObjectMetadataRequest createGetObjectMetadataRequest(final ProcessContext context, final Map<String, String> attributes) {