[FLINK-4559][kinesis-connector] Fix AWSUtil.getCredentialsProvider() to not return null
Return a DefaultAWSCredentialsProviderChain instead of null when AWS_CREDENTIALS_PROVIDER config is set to "AUTO" This closes #2470 Project: http://git-wip-us.apache.org/repos/asf/flink/repo Commit: http://git-wip-us.apache.org/repos/asf/flink/commit/cab76f6e Tree: http://git-wip-us.apache.org/repos/asf/flink/tree/cab76f6e Diff: http://git-wip-us.apache.org/repos/asf/flink/diff/cab76f6e Branch: refs/heads/flip-6 Commit: cab76f6e23f6b491ecde5856dd32e4de95a0af90 Parents: 87aea72 Author: Tzu-Li (Gordon) Tai <tzuli...@apache.org> Authored: Sun Sep 4 01:37:39 2016 +0800 Committer: Tzu-Li (Gordon) Tai <tzuli...@apache.org> Committed: Tue Sep 6 20:12:34 2016 +0800 ---------------------------------------------------------------------- .../streaming/connectors/kinesis/util/AWSUtil.java | 12 ++++-------- 1 file changed, 4 insertions(+), 8 deletions(-) ---------------------------------------------------------------------- http://git-wip-us.apache.org/repos/asf/flink/blob/cab76f6e/flink-streaming-connectors/flink-connector-kinesis/src/main/java/org/apache/flink/streaming/connectors/kinesis/util/AWSUtil.java ---------------------------------------------------------------------- diff --git a/flink-streaming-connectors/flink-connector-kinesis/src/main/java/org/apache/flink/streaming/connectors/kinesis/util/AWSUtil.java b/flink-streaming-connectors/flink-connector-kinesis/src/main/java/org/apache/flink/streaming/connectors/kinesis/util/AWSUtil.java index 06f967b..81c0b6b 100644 --- a/flink-streaming-connectors/flink-connector-kinesis/src/main/java/org/apache/flink/streaming/connectors/kinesis/util/AWSUtil.java +++ b/flink-streaming-connectors/flink-connector-kinesis/src/main/java/org/apache/flink/streaming/connectors/kinesis/util/AWSUtil.java @@ -22,6 +22,7 @@ import com.amazonaws.ClientConfigurationFactory; import com.amazonaws.auth.AWSCredentials; import com.amazonaws.auth.AWSCredentialsProvider; import com.amazonaws.auth.BasicAWSCredentials; +import com.amazonaws.auth.DefaultAWSCredentialsProviderChain; import com.amazonaws.auth.EnvironmentVariableCredentialsProvider; import com.amazonaws.auth.SystemPropertiesCredentialsProvider; import com.amazonaws.auth.profile.ProfileCredentialsProvider; @@ -50,13 +51,8 @@ public class AWSUtil { awsClientConfig.setUserAgent("Apache Flink " + EnvironmentInformation.getVersion() + " (" + EnvironmentInformation.getRevisionInformation().commitId + ") Kinesis Connector"); - AmazonKinesisClient client; - if (AWSUtil.getCredentialsProvider(configProps) != null) { - client = new AmazonKinesisClient( - AWSUtil.getCredentialsProvider(configProps).getCredentials(), awsClientConfig); - } else { - client = new AmazonKinesisClient(awsClientConfig); - } + AmazonKinesisClient client = new AmazonKinesisClient( + AWSUtil.getCredentialsProvider(configProps).getCredentials(), awsClientConfig); client.setRegion(Region.getRegion(Regions.fromName(configProps.getProperty(AWSConfigConstants.AWS_REGION)))); if (configProps.containsKey(AWSConfigConstants.AWS_ENDPOINT)) { @@ -94,7 +90,7 @@ public class AWSUtil { : new ProfileCredentialsProvider(profileConfigPath, profileName); break; case AUTO: - credentialsProvider = null; + credentialsProvider = new DefaultAWSCredentialsProviderChain(); break; default: case BASIC: