vahmed-hamdy commented on a change in pull request #18603:
URL: https://github.com/apache/flink/pull/18603#discussion_r803864907



##########
File path: 
flink-connectors/flink-connector-aws-kinesis-firehose/src/main/java/org/apache/flink/connector/firehose/table/KinesisFirehoseDynamicSink.java
##########
@@ -0,0 +1,185 @@
+/*
+ * 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.firehose.table;
+
+import org.apache.flink.annotation.Internal;
+import org.apache.flink.api.common.serialization.SerializationSchema;
+import org.apache.flink.connector.base.table.sink.AsyncDynamicTableSink;
+import org.apache.flink.connector.base.table.sink.AsyncDynamicTableSinkBuilder;
+import org.apache.flink.connector.firehose.sink.KinesisFirehoseSink;
+import org.apache.flink.connector.firehose.sink.KinesisFirehoseSinkBuilder;
+import org.apache.flink.table.connector.ChangelogMode;
+import org.apache.flink.table.connector.format.EncodingFormat;
+import org.apache.flink.table.connector.sink.DynamicTableSink;
+import org.apache.flink.table.connector.sink.SinkProvider;
+import org.apache.flink.table.data.RowData;
+import org.apache.flink.table.types.DataType;
+import org.apache.flink.util.Preconditions;
+
+import software.amazon.awssdk.services.firehose.model.Record;
+
+import javax.annotation.Nullable;
+
+import java.util.Optional;
+import java.util.Properties;
+
+/** Kinesis-firehose-backed {@link AsyncDynamicTableSink}. */
+@Internal
+public class KinesisFirehoseDynamicSink extends AsyncDynamicTableSink<Record> {
+
+    /** Consumed data type of the table. */
+    private final DataType consumedDataType;
+
+    /** The Kinesis stream to write to. */
+    private final String deliveryStream;
+
+    /** Properties for the Firehose DataStream Sink. */
+    private final Properties firehoseClientProperties;
+
+    /** Sink format for encoding records to Kinesis. */
+    private final EncodingFormat<SerializationSchema<RowData>> encodingFormat;
+
+    private final Boolean failOnError;
+
+    protected KinesisFirehoseDynamicSink(
+            @Nullable Integer maxBatchSize,
+            @Nullable Integer maxInFlightRequests,
+            @Nullable Integer maxBufferedRequests,
+            @Nullable Long maxBufferSizeInBytes,
+            @Nullable Long maxTimeInBufferMS,
+            @Nullable Boolean failOnError,
+            @Nullable DataType consumedDataType,
+            String deliveryStream,
+            @Nullable Properties firehoseClientProperties,
+            EncodingFormat<SerializationSchema<RowData>> encodingFormat) {
+        super(
+                maxBatchSize,
+                maxInFlightRequests,
+                maxBufferedRequests,
+                maxBufferSizeInBytes,
+                maxTimeInBufferMS);
+        this.failOnError = failOnError;
+        this.firehoseClientProperties = firehoseClientProperties;
+        this.consumedDataType =
+                Preconditions.checkNotNull(consumedDataType, "Consumed data 
type must not be null");
+        this.deliveryStream =
+                Preconditions.checkNotNull(
+                        deliveryStream, "Firehose Delivery stream name must 
not be null");
+        this.encodingFormat =
+                Preconditions.checkNotNull(encodingFormat, "Encoding format 
must not be null");
+    }
+
+    @Override
+    public ChangelogMode getChangelogMode(ChangelogMode requestedMode) {
+        return encodingFormat.getChangelogMode();
+    }
+
+    @Override
+    public SinkRuntimeProvider getSinkRuntimeProvider(Context context) {
+        SerializationSchema<RowData> serializationSchema =
+                encodingFormat.createRuntimeEncoder(context, consumedDataType);
+
+        KinesisFirehoseSinkBuilder<RowData> builder =
+                KinesisFirehoseSink.<RowData>builder()
+                        .setSerializationSchema(serializationSchema)
+                        .setFirehoseClientProperties(firehoseClientProperties)
+                        .setDeliveryStreamName(deliveryStream);
+
+        Optional.ofNullable(failOnError).ifPresent(builder::setFailOnError);
+        super.addAsyncOptionsToSinkBuilder(builder);
+
+        KinesisFirehoseSink<RowData> kdsSink = builder.build();
+        return SinkProvider.of(kdsSink);
+    }
+
+    @Override
+    public DynamicTableSink copy() {
+        return new KinesisFirehoseDynamicSink(
+                maxBatchSize,
+                maxInFlightRequests,
+                maxBufferedRequests,
+                maxBufferSizeInBytes,
+                maxTimeInBufferMS,
+                failOnError,
+                consumedDataType,
+                deliveryStream,
+                firehoseClientProperties,
+                encodingFormat);
+    }
+
+    @Override
+    public String asSummaryString() {
+        return "firehose";

Review comment:
       I only found usages in the scala code base, It is implemented in other 
connectors though (i.e. kafka, legacy kinesis)




-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: [email protected]

For queries about this service, please contact Infrastructure at:
[email protected]


Reply via email to