Skip to content

Commit

Permalink
Deprecate AWS IOs (Java) using AWS SDK v1 in favor of IOs in amazon-w…
Browse files Browse the repository at this point in the history
…eb-services2 (#22093)
  • Loading branch information
Moritz Mack committed Jul 8, 2022
1 parent 0e14f51 commit f5435c0
Show file tree
Hide file tree
Showing 7 changed files with 69 additions and 23 deletions.
5 changes: 4 additions & 1 deletion CHANGES.md
Original file line number Diff line number Diff line change
Expand Up @@ -74,7 +74,10 @@
## Deprecations

* Support for Spark 2.4.x is deprecated and will be dropped with the release of Beam 2.44.0 or soon after (Spark runner) ([#22094](https://github.com/apache/beam/issues/22094)).
* X behavior is deprecated and will be removed in X versions ([#X](https://github.com/apache/beam/issues/X)).
* The modules [amazon-web-services](https://github.com/apache/beam/tree/master/sdks/java/io/amazon-web-services) and
[kinesis](https://github.com/apache/beam/tree/master/sdks/java/io/kinesis) for AWS Java SDK v1 are deprecated
in favor of [amazon-web-services2](https://github.com/apache/beam/tree/master/sdks/java/io/amazon-web-services2)
and will be eventually removed after a few Beam releases (Java) ([#21249](https://github.com/apache/beam/issues/21249)).

## Bugfixes

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -131,8 +131,13 @@
* <li>ScanRequestFn, which you build a ScanRequest object with at least table name and total
* number of segment. Note This number should base on the number of your workers
* </ul>
*
* @deprecated Module <code>beam-sdks-java-io-amazon-web-services</code> is deprecated and will be
* eventually removed. Please migrate to {@link org.apache.beam.sdk.io.aws2.dynamodb.DynamoDBIO}
* in module <code>beam-sdks-java-io-amazon-web-services2</code>.
*/
@Experimental(Kind.SOURCE_SINK)
@Deprecated
@SuppressWarnings({
"nullness" // TODO(https://github.com/apache/beam/issues/20497)
})
Expand Down Expand Up @@ -227,6 +232,11 @@ public Read<T> withCoder(Coder<T> coder) {

@Override
public PCollection<T> expand(PBegin input) {
LoggerFactory.getLogger(DynamoDBIO.class)
.warn(
"You are using a deprecated IO for DynamoDB. Please migrate to module "
+ "'org.apache.beam:beam-sdks-java-io-amazon-web-services2'.");

checkArgument((getScanRequestFn() != null), "withScanRequestFn() is required");
checkArgument((getAwsClientsProvider() != null), "withAwsClientsProvider() is required");
ScanRequest scanRequest = getScanRequestFn().apply(null);
Expand Down Expand Up @@ -454,6 +464,11 @@ public Write<T> withDeduplicateKeys(List<String> deduplicateKeys) {

@Override
public PCollection<Void> expand(PCollection<T> input) {
LoggerFactory.getLogger(DynamoDBIO.class)
.warn(
"You are using a deprecated IO for DynamoDB. Please migrate to module "
+ "'org.apache.beam:beam-sdks-java-io-amazon-web-services2'.");

return input.apply(ParDo.of(new WriteFn<>(this)));
}

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -86,7 +86,11 @@
* {@link FileSystem} implementation for storage systems that use the S3 protocol.
*
* @see S3FileSystemSchemeRegistrar
* @deprecated Module <code>beam-sdks-java-io-amazon-web-services</code> is deprecated and will be
* eventually removed. Please migrate to module <code>beam-sdks-java-io-amazon-web-services2
* </code>.
*/
@Deprecated
@SuppressWarnings({
"nullness" // TODO(https://github.com/apache/beam/issues/20497)
})
Expand Down Expand Up @@ -120,6 +124,10 @@ class S3FileSystem extends FileSystem<S3ResourceId> {
MoreExecutors.listeningDecorator(
Executors.newFixedThreadPool(
config.getS3ThreadPoolSize(), new ThreadFactoryBuilder().setDaemon(true).build()));

LOG.warn(
"You are using a deprecated file system for S3. Please migrate to module "
+ "'org.apache.beam:beam-sdks-java-io-amazon-web-services2'.");
}

S3FileSystem(S3Options options) {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -85,8 +85,13 @@
* If you need the full ResponseMetadata and SdkHttpMetadata you can call {@link
* Write#withFullPublishResult}. If you need the HTTP status code but not the response headers you
* can call {@link Write#withFullPublishResultWithoutHeaders}.
*
* @deprecated Module <code>beam-sdks-java-io-amazon-web-services</code> is deprecated and will be
* eventually removed. Please migrate to {@link org.apache.beam.sdk.io.aws2.sns.SnsIO} in module
* <code>beam-sdks-java-io-amazon-web-services2</code>.
*/
@Experimental(Kind.SOURCE_SINK)
@Deprecated
@SuppressWarnings({
"nullness" // TODO(https://github.com/apache/beam/issues/20497)
})
Expand Down Expand Up @@ -308,6 +313,11 @@ public Write withCoder(Coder<PublishResult> coder) {

@Override
public PCollectionTuple expand(PCollection<PublishRequest> input) {
LoggerFactory.getLogger(SnsIO.class)
.warn(
"You are using a deprecated IO for Sns. Please migrate to module "
+ "'org.apache.beam:beam-sdks-java-io-amazon-web-services2'.");

checkArgument(getTopicName() != null, "withTopicName() is required");
PCollectionTuple result =
input.apply(
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -36,6 +36,7 @@
import org.apache.beam.sdk.values.PDone;
import org.checkerframework.checker.nullness.qual.Nullable;
import org.joda.time.Duration;
import org.slf4j.LoggerFactory;

/**
* An unbounded source for Amazon Simple Queue Service (SQS).
Expand Down Expand Up @@ -77,8 +78,13 @@
* }</pre>
*
* <p>For more information on the available options see {@link AwsOptions}.
*
* @deprecated Module <code>beam-sdks-java-io-amazon-web-services</code> is deprecated and will be
* eventually removed. Please migrate to {@link org.apache.beam.sdk.io.aws2.sqs.SqsIO} in module
* <code>beam-sdks-java-io-amazon-web-services2</code>.
*/
@Experimental(Kind.SOURCE_SINK)
@Deprecated
@SuppressWarnings({
"nullness" // TODO(https://github.com/apache/beam/issues/20497)
})
Expand Down Expand Up @@ -164,6 +170,10 @@ public Read withQueueUrl(String queueUrl) {

@Override
public PCollection<Message> expand(PBegin input) {
LoggerFactory.getLogger(SqsIO.class)
.warn(
"You are using a deprecated IO for Sqs. Please migrate to module "
+ "'org.apache.beam:beam-sdks-java-io-amazon-web-services2'.");

org.apache.beam.sdk.io.Read.Unbounded<Message> unbounded =
org.apache.beam.sdk.io.Read.from(
Expand Down Expand Up @@ -197,6 +207,11 @@ abstract static class Builder {

@Override
public PDone expand(PCollection<SendMessageRequest> input) {
LoggerFactory.getLogger(SqsIO.class)
.warn(
"You are using a deprecated IO for Sqs. Please migrate to module "
+ "'org.apache.beam:beam-sdks-java-io-amazon-web-services2'.");

input.apply(
ParDo.of(
new SqsWriteFn(
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -314,11 +314,16 @@
* <p>For more information about configuratiom parameters, see the <a
* href="https://github.com/awslabs/amazon-kinesis-producer/blob/master/java/amazon-kinesis-producer-sample/default_config.properties">sample
* of configuration file</a>.
*
* @deprecated Module <code>beam-sdks-java-io-kinesis</code> is deprecated and will be eventually
* removed. Please migrate to {@link org.apache.beam.sdk.io.aws2.kinesis.KinesisIO} in module
* <code>beam-sdks-java-io-amazon-web-services2</code>.
*/
@Experimental(Kind.SOURCE_SINK)
@SuppressWarnings({
"nullness" // TODO(https://github.com/apache/beam/issues/20497)
})
@Deprecated
public final class KinesisIO {

private static final Logger LOG = LoggerFactory.getLogger(KinesisIO.class);
Expand Down Expand Up @@ -655,6 +660,10 @@ public Read<T> withMaxCapacityPerShard(Integer maxCapacity) {

@Override
public PCollection<T> expand(PBegin input) {
LOG.warn(
"You are using a deprecated IO for Kinesis. Please migrate to module "
+ "'org.apache.beam:beam-sdks-java-io-amazon-web-services2'.");

Unbounded<KinesisRecord> unbounded =
org.apache.beam.sdk.io.Read.from(
new KinesisSource(
Expand Down Expand Up @@ -875,6 +884,10 @@ Write withRetries(int retries) {

@Override
public PDone expand(PCollection<byte[]> input) {
LOG.warn(
"You are using a deprecated IO for Kinesis. Please migrate to module "
+ "'org.apache.beam:beam-sdks-java-io-amazon-web-services2'.");

checkArgument(getStreamName() != null, "withStreamName() is required");
checkArgument(
(getPartitionKey() != null) || (getPartitioner() != null),
Expand Down
26 changes: 4 additions & 22 deletions website/www/site/data/io_matrix.yaml
Original file line number Diff line number Diff line change
Expand Up @@ -136,11 +136,8 @@ categories:
description: "`FileSystem` implementation for [Amazon S3](https://aws.amazon.com/s3/)."
implementations:
- language: java
name: org.apache.beam.sdk.io.aws2.s3.S3FileSystemRegistrar (recommended)
name: org.apache.beam.sdk.io.aws2.s3.S3FileSystemRegistrar
url: https://beam.apache.org/releases/javadoc/current/org/apache/beam/sdk/io/aws2/s3/S3FileSystemRegistrar.html
- language: java
name: org.apache.beam.sdk.io.aws.s3.S3FileSystemRegistrar
url: https://beam.apache.org/releases/javadoc/current/org/apache/beam/sdk/io/aws/s3/S3FileSystemRegistrar.html
- transform: In-memory
description: "`FileSystem` implementation in memory; useful for testing."
implementations:
Expand All @@ -154,11 +151,8 @@ categories:
description: PTransforms for reading from and writing to [Kinesis](https://aws.amazon.com/kinesis/) streams.
implementations:
- language: java
name: org.apache.beam.sdk.io.aws2.kinesis.KinesisIO (recommended)
name: org.apache.beam.sdk.io.aws2.kinesis.KinesisIO
url: https://beam.apache.org/releases/javadoc/current/org/apache/beam/sdk/io/aws2/kinesis/KinesisIO.html
- language: java
name: org.apache.beam.sdk.io.kinesis.KinesisIO
url: https://beam.apache.org/releases/javadoc/current/org/apache/beam/sdk/io/kinesis/KinesisIO.html
- transform: AmqpIO
description: AMQP 1.0 protocol using the Apache QPid Proton-J library
implementations:
Expand Down Expand Up @@ -211,20 +205,11 @@ categories:
description: An unbounded source for [Amazon Simple Queue Service (SQS)](https://aws.amazon.com/sqs/).
implementations:
- language: java
name: org.apache.beam.sdk.io.aws2.sqs.SqsIO (recommended)
name: org.apache.beam.sdk.io.aws2.sqs.SqsIO
url: https://beam.apache.org/releases/javadoc/current/org/apache/beam/sdk/io/aws2/sqs/SqsIO.html
- language: java
name: org.apache.beam.sdk.io.aws.sqs.SqsIO
url: https://beam.apache.org/releases/javadoc/current/org/apache/beam/sdk/io/aws/sqs/SqsIO.html
- transform: SnsIO
description: PTransforms for writing to [Amazon Simple Notification Service (SNS)](https://aws.amazon.com/sns/).
implementations:
- language: java
name: org.apache.beam.sdk.io.aws2.sns.SnsIO (recommended)
url: https://beam.apache.org/releases/javadoc/current/org/apache/beam/sdk/io/aws2/sns/SnsIO.html
- language: java
name: org.apache.beam.sdk.io.aws.sns.SnsIO
url: https://beam.apache.org/releases/javadoc/current/org/apache/beam/sdk/io/aws/sns/SnsIO.html
- language: java
name: org.apache.beam.sdk.io.aws2.sns.SnsIO
url: https://beam.apache.org/releases/javadoc/current/org/apache/beam/sdk/io/aws2/sns/SnsIO.html
Expand Down Expand Up @@ -359,11 +344,8 @@ categories:
description: Read from and write to [Amazon DynamoDB](https://aws.amazon.com/dynamodb/).
implementations:
- language: java
name: org.apache.beam.sdk.io.aws2.dynamodb.DynamoDBIO (recommended)
name: org.apache.beam.sdk.io.aws2.dynamodb.DynamoDBIO
url: https://beam.apache.org/releases/javadoc/current/org/apache/beam/sdk/io/aws2/dynamodb/DynamoDBIO.html
- language: java
name: org.apache.beam.sdk.io.aws.dynamodb.DynamoDBIO
url: https://beam.apache.org/releases/javadoc/current/org/apache/beam/sdk/io/aws/dynamodb/DynamoDBIO.html
- transform: ClickHouseIO
description: Transform for writing to [ClickHouse](https://clickhouse.tech/).
implementations:
Expand Down

0 comments on commit f5435c0

Please sign in to comment.