You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@flink.apache.org by da...@apache.org on 2022/11/08 22:26:08 UTC
[flink-connector-aws] branch main updated: [FLINK-29937][Connectors/DynamoDB] Enhanced DynamoDB Element Converter
This is an automated email from the ASF dual-hosted git repository.
dannycranmer pushed a commit to branch main
in repository https://gitbox.apache.org/repos/asf/flink-connector-aws.git
The following commit(s) were added to refs/heads/main by this push:
new a28dc94 [FLINK-29937][Connectors/DynamoDB] Enhanced DynamoDB Element Converter
a28dc94 is described below
commit a28dc943fafb2c32c287f5e5948396ddfea224e8
Author: Danny Cranmer <da...@apache.org>
AuthorDate: Tue Nov 8 20:34:43 2022 +0000
[FLINK-29937][Connectors/DynamoDB] Enhanced DynamoDB Element Converter
---
flink-connector-dynamodb/pom.xml | 4 ++
.../sink/DynamoDBEnhancedElementConverter.java | 71 ++++++++++++++++++++
.../sink/DynamoDBEnhancedElementConverterTest.java | 77 ++++++++++++++++++++++
.../dynamodb/sink/examples/SinkIntoDynamoDb.java | 2 +-
...kIntoDynamoDbUsingEnhancedElementConverter.java | 75 +++++++++++++++++++++
.../flink/connector/dynamodb/util/Order.java | 46 +++++++++++++
.../src/main/resources/META-INF/NOTICE | 1 +
7 files changed, 275 insertions(+), 1 deletion(-)
diff --git a/flink-connector-dynamodb/pom.xml b/flink-connector-dynamodb/pom.xml
index 5fad9c4..740e4d2 100644
--- a/flink-connector-dynamodb/pom.xml
+++ b/flink-connector-dynamodb/pom.xml
@@ -72,6 +72,10 @@ under the License.
<groupId>software.amazon.awssdk</groupId>
<artifactId>dynamodb</artifactId>
</dependency>
+ <dependency>
+ <groupId>software.amazon.awssdk</groupId>
+ <artifactId>dynamodb-enhanced</artifactId>
+ </dependency>
<!-- Test dependencies -->
<dependency>
diff --git a/flink-connector-dynamodb/src/main/java/org/apache/flink/connector/dynamodb/sink/DynamoDBEnhancedElementConverter.java b/flink-connector-dynamodb/src/main/java/org/apache/flink/connector/dynamodb/sink/DynamoDBEnhancedElementConverter.java
new file mode 100644
index 0000000..2b3053f
--- /dev/null
+++ b/flink-connector-dynamodb/src/main/java/org/apache/flink/connector/dynamodb/sink/DynamoDBEnhancedElementConverter.java
@@ -0,0 +1,71 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements. See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.flink.connector.dynamodb.sink;
+
+import org.apache.flink.api.connector.sink2.SinkWriter;
+import org.apache.flink.connector.base.sink.writer.ElementConverter;
+
+import software.amazon.awssdk.enhanced.dynamodb.mapper.BeanTableSchema;
+import software.amazon.awssdk.enhanced.dynamodb.mapper.annotations.DynamoDbBean;
+
+/**
+ * A generic {@link ElementConverter} that uses the dynamodb-enhanced client to build a {@link
+ * DynamoDbWriteRequest} from a POJO annotated with {@link DynamoDbBean}.
+ *
+ * @param <InputT> The
+ */
+public class DynamoDBEnhancedElementConverter<InputT>
+ implements ElementConverter<InputT, DynamoDbWriteRequest> {
+
+ private static final long serialVersionUID = 1L;
+
+ private final Class<InputT> recordType;
+ private final boolean ignoreNulls;
+ private transient BeanTableSchema<InputT> tableSchema;
+
+ public DynamoDBEnhancedElementConverter(final Class<InputT> recordType) {
+ this(recordType, false);
+ }
+
+ public DynamoDBEnhancedElementConverter(
+ final Class<InputT> recordType, final boolean ignoreNulls) {
+ this.recordType = recordType;
+ this.ignoreNulls = ignoreNulls;
+
+ // Attempt to create a table schema now to bubble up errors before starting job
+ createTableSchema(recordType);
+ }
+
+ @Override
+ public DynamoDbWriteRequest apply(InputT element, SinkWriter.Context context) {
+ if (tableSchema == null) {
+ // We have to lazily initialise this because BeanTableSchema is not serializable and
+ // there is no open() method
+ tableSchema = createTableSchema(recordType);
+ }
+
+ return new DynamoDbWriteRequest.Builder()
+ .setType(DynamoDbWriteRequestType.PUT)
+ .setItem(tableSchema.itemToMap(element, ignoreNulls))
+ .build();
+ }
+
+ private BeanTableSchema<InputT> createTableSchema(final Class<InputT> recordType) {
+ return BeanTableSchema.create(recordType);
+ }
+}
diff --git a/flink-connector-dynamodb/src/test/java/org/apache/flink/connector/dynamodb/sink/DynamoDBEnhancedElementConverterTest.java b/flink-connector-dynamodb/src/test/java/org/apache/flink/connector/dynamodb/sink/DynamoDBEnhancedElementConverterTest.java
new file mode 100644
index 0000000..57fb55b
--- /dev/null
+++ b/flink-connector-dynamodb/src/test/java/org/apache/flink/connector/dynamodb/sink/DynamoDBEnhancedElementConverterTest.java
@@ -0,0 +1,77 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.flink.connector.dynamodb.sink;
+
+import org.apache.flink.connector.base.sink.writer.ElementConverter;
+import org.apache.flink.connector.dynamodb.util.Order;
+
+import org.junit.jupiter.api.Test;
+
+import static org.apache.flink.connector.dynamodb.sink.DynamoDbWriteRequestType.PUT;
+import static org.assertj.core.api.Assertions.assertThat;
+import static org.assertj.core.api.Assertions.assertThatExceptionOfType;
+
+class DynamoDBEnhancedElementConverterTest {
+
+ @Test
+ void testBadType() {
+ assertThatExceptionOfType(IllegalArgumentException.class)
+ .isThrownBy(() -> new DynamoDBEnhancedElementConverter<>(Integer.class))
+ .withMessageContaining(
+ "A DynamoDb bean class must be annotated with @DynamoDbBean");
+ }
+
+ @Test
+ void testConvertOrderToDynamoDbWriteRequest() {
+ ElementConverter<Order, DynamoDbWriteRequest> elementConverter =
+ new DynamoDBEnhancedElementConverter<>(Order.class);
+ Order order = new Order("orderId", 1, 2.0);
+
+ DynamoDbWriteRequest actual = elementConverter.apply(order, null);
+
+ assertThat(actual.getType()).isEqualTo(PUT);
+ assertThat(actual.getItem()).containsOnlyKeys("orderId", "quantity", "total");
+ assertThat(actual.getItem().get("orderId").s()).isEqualTo("orderId");
+ assertThat(actual.getItem().get("quantity").n()).isEqualTo("1");
+ assertThat(actual.getItem().get("total").n()).isEqualTo("2.0");
+ }
+
+ @Test
+ void testConvertOrderToDynamoDbWriteRequestWithIgnoresNull() {
+ ElementConverter<Order, DynamoDbWriteRequest> elementConverter =
+ new DynamoDBEnhancedElementConverter<>(Order.class, true);
+ Order order = new Order(null, 1, 2.0);
+
+ DynamoDbWriteRequest actual = elementConverter.apply(order, null);
+
+ assertThat(actual.getItem()).containsOnlyKeys("quantity", "total");
+ }
+
+ @Test
+ void testConvertOrderToDynamoDbWriteRequestWritesNull() {
+ ElementConverter<Order, DynamoDbWriteRequest> elementConverter =
+ new DynamoDBEnhancedElementConverter<>(Order.class, false);
+ Order order = new Order(null, 1, 2.0);
+
+ DynamoDbWriteRequest actual = elementConverter.apply(order, null);
+
+ assertThat(actual.getItem()).containsOnlyKeys("orderId", "quantity", "total");
+ assertThat(actual.getItem().get("orderId").nul()).isTrue();
+ }
+}
diff --git a/flink-connector-dynamodb/src/test/java/org/apache/flink/connector/dynamodb/sink/examples/SinkIntoDynamoDb.java b/flink-connector-dynamodb/src/test/java/org/apache/flink/connector/dynamodb/sink/examples/SinkIntoDynamoDb.java
index 11726c9..753f5ee 100644
--- a/flink-connector-dynamodb/src/test/java/org/apache/flink/connector/dynamodb/sink/examples/SinkIntoDynamoDb.java
+++ b/flink-connector-dynamodb/src/test/java/org/apache/flink/connector/dynamodb/sink/examples/SinkIntoDynamoDb.java
@@ -60,7 +60,7 @@ public class SinkIntoDynamoDb {
fromGen.map(new TestRequestMapper()).sinkTo(dynamoDbSink);
- env.execute("DynamoDb Async Sink Example Program");
+ env.execute("DynamoDb Sink Example Job");
}
/** Example DynamoDB request attributes mapper. */
diff --git a/flink-connector-dynamodb/src/test/java/org/apache/flink/connector/dynamodb/sink/examples/SinkIntoDynamoDbUsingEnhancedElementConverter.java b/flink-connector-dynamodb/src/test/java/org/apache/flink/connector/dynamodb/sink/examples/SinkIntoDynamoDbUsingEnhancedElementConverter.java
new file mode 100644
index 0000000..f255e8e
--- /dev/null
+++ b/flink-connector-dynamodb/src/test/java/org/apache/flink/connector/dynamodb/sink/examples/SinkIntoDynamoDbUsingEnhancedElementConverter.java
@@ -0,0 +1,75 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.flink.connector.dynamodb.sink.examples;
+
+import org.apache.flink.api.common.functions.RichMapFunction;
+import org.apache.flink.connector.aws.config.AWSConfigConstants;
+import org.apache.flink.connector.dynamodb.sink.DynamoDBEnhancedElementConverter;
+import org.apache.flink.connector.dynamodb.sink.DynamoDbSink;
+import org.apache.flink.connector.dynamodb.util.Order;
+import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment;
+
+import org.apache.commons.math3.random.RandomDataGenerator;
+
+import java.util.Properties;
+import java.util.UUID;
+
+/**
+ * An example application demonstrating how to use the {@link DynamoDbSink} to sink into DynamoDb
+ * using the {@link DynamoDBEnhancedElementConverter}.
+ */
+public class SinkIntoDynamoDbUsingEnhancedElementConverter {
+
+ private static final String DYNAMODB_TABLE = "orders";
+ private static final String REGION = "us-east-1";
+
+ public static void main(String[] args) throws Exception {
+ final StreamExecutionEnvironment env = StreamExecutionEnvironment.getExecutionEnvironment();
+
+ Properties sinkProperties = new Properties();
+ sinkProperties.put(AWSConfigConstants.AWS_REGION, REGION);
+
+ DynamoDbSink<Order> dynamoDbSink =
+ DynamoDbSink.<Order>builder()
+ .setDestinationTableName(DYNAMODB_TABLE)
+ .setElementConverter(new DynamoDBEnhancedElementConverter<>(Order.class))
+ .setDynamoDbProperties(sinkProperties)
+ .build();
+
+ env.fromSequence(1, 1_000L)
+ .map(new TestRequestMapper())
+ .returns(Order.class)
+ .sinkTo(dynamoDbSink);
+
+ env.execute("DynamoDb Sink Example Job");
+ }
+
+ /** Example RichMapFunction to generate Order from String. */
+ public static class TestRequestMapper extends RichMapFunction<Long, Order> {
+ private final RandomDataGenerator random = new RandomDataGenerator();
+
+ @Override
+ public Order map(final Long i) throws Exception {
+ return new Order(
+ UUID.randomUUID().toString(),
+ random.nextInt(0, 100),
+ random.getRandomGenerator().nextDouble() * 1000);
+ }
+ }
+}
diff --git a/flink-connector-dynamodb/src/test/java/org/apache/flink/connector/dynamodb/util/Order.java b/flink-connector-dynamodb/src/test/java/org/apache/flink/connector/dynamodb/util/Order.java
new file mode 100644
index 0000000..81e651c
--- /dev/null
+++ b/flink-connector-dynamodb/src/test/java/org/apache/flink/connector/dynamodb/util/Order.java
@@ -0,0 +1,46 @@
+package org.apache.flink.connector.dynamodb.util;
+
+import org.apache.flink.connector.dynamodb.sink.DynamoDBEnhancedElementConverter;
+
+import software.amazon.awssdk.enhanced.dynamodb.mapper.annotations.DynamoDbBean;
+
+/** A test {@link DynamoDbBean} POJO for use with {@link DynamoDBEnhancedElementConverter}. */
+@DynamoDbBean
+public class Order {
+
+ private String orderId;
+ private int quantity;
+ private double total;
+
+ public Order() {}
+
+ public Order(String orderId, int quantity, double total) {
+ this.orderId = orderId;
+ this.quantity = quantity;
+ this.total = total;
+ }
+
+ public String getOrderId() {
+ return orderId;
+ }
+
+ public void setOrderId(String orderId) {
+ this.orderId = orderId;
+ }
+
+ public int getQuantity() {
+ return quantity;
+ }
+
+ public void setQuantity(int quantity) {
+ this.quantity = quantity;
+ }
+
+ public double getTotal() {
+ return total;
+ }
+
+ public void setTotal(double total) {
+ this.total = total;
+ }
+}
diff --git a/flink-sql-connector-dynamodb/src/main/resources/META-INF/NOTICE b/flink-sql-connector-dynamodb/src/main/resources/META-INF/NOTICE
index d2e2b48..89e08bd 100644
--- a/flink-sql-connector-dynamodb/src/main/resources/META-INF/NOTICE
+++ b/flink-sql-connector-dynamodb/src/main/resources/META-INF/NOTICE
@@ -8,6 +8,7 @@ The Apache Software Foundation (http://www.apache.org/).
This project bundles the following dependencies under the Apache Software License 2.0. (http://www.apache.org/licenses/LICENSE-2.0.txt)
- software.amazon.awssdk:dynamodb:2.17.247
+- software.amazon.awssdk:dynamodb-enhanced:2.17.247
- software.amazon.awssdk:aws-json-protocol:2.17.247
- software.amazon.awssdk:protocol-core:2.17.247
- software.amazon.awssdk:profiles:2.17.247