Github user rmetzger commented on a diff in the pull request:

    https://github.com/apache/flink/pull/1911#discussion_r60733595
  
    --- Diff: 
flink-streaming-connectors/flink-connector-kinesis/src/main/java/org/apache/flink/streaming/connectors/kinesis/FlinkKinesisConsumer.java
 ---
    @@ -0,0 +1,481 @@
    +/*
    + * 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.streaming.connectors.kinesis;
    +
    +import com.amazonaws.regions.Regions;
    +import org.apache.flink.api.common.typeinfo.TypeInformation;
    +import org.apache.flink.api.java.typeutils.ResultTypeQueryable;
    +import org.apache.flink.configuration.Configuration;
    +import org.apache.flink.runtime.state.CheckpointListener;
    +import 
org.apache.flink.streaming.api.checkpoint.CheckpointedAsynchronously;
    +import 
org.apache.flink.streaming.api.functions.source.RichParallelSourceFunction;
    +import org.apache.flink.streaming.api.watermark.Watermark;
    +import 
org.apache.flink.streaming.connectors.kinesis.config.KinesisConfigConstants;
    +import 
org.apache.flink.streaming.connectors.kinesis.internals.KinesisDataFetcher;
    +import 
org.apache.flink.streaming.connectors.kinesis.config.CredentialProviderType;
    +import 
org.apache.flink.streaming.connectors.kinesis.config.InitialPosition;
    +import 
org.apache.flink.streaming.connectors.kinesis.model.KinesisStreamShard;
    +import 
org.apache.flink.streaming.connectors.kinesis.model.SentinelSequenceNumber;
    +import org.apache.flink.streaming.connectors.kinesis.proxy.KinesisProxy;
    +import 
org.apache.flink.streaming.connectors.kinesis.serialization.KinesisDeserializationSchemaWrapper;
    +import 
org.apache.flink.streaming.connectors.kinesis.serialization.KinesisDeserializationSchema;
    +import org.apache.flink.streaming.connectors.kinesis.util.AWSUtil;
    +import org.apache.flink.streaming.util.serialization.DeserializationSchema;
    +import org.slf4j.Logger;
    +import org.slf4j.LoggerFactory;
    +
    +import java.io.IOException;
    +import java.util.Collections;
    +import java.util.Map;
    +import java.util.HashMap;
    +import java.util.List;
    +import java.util.ArrayList;
    +import java.util.Properties;
    +
    +import static com.google.common.base.Preconditions.checkArgument;
    +import static com.google.common.base.Preconditions.checkNotNull;
    +
    +/**
    + * The Flink Kinesis Consumer is a parallel streaming data source that 
pulls data from multiple AWS Kinesis streams
    + * within the same AWS service region. Each instance of the consumer is 
responsible for fetching data records from
    + * one or more Kinesis shards.
    + *
    + * <p>To leverage Flink's checkpointing mechanics for exactly-once 
streaming processing guarantees, the Flink Kinesis
    + * consumer is implemented with the AWS Java SDK, instead of the 
officially recommended AWS Kinesis Client Library, for
    + * low-level control on the management of stream state. The Flink Kinesis 
Connector also supports setting the initial
    + * starting points of Kinesis streams, namely TRIM_HORIZON and LATEST:</p>
    + *
    + * <ul>
    + * </ul>
    + *
    + * <p><b>NOTE:</b> The current implementation does not correctly handle 
resharding of AWS Kinesis streams.</p>
    + * <p><b>NOTE:</b> Since Kinesis and Kafka share many common abstractions, 
the implementation is heavily based on
    + * the Flink Kafka Consumer.</p>
    + *
    + * @param <T> the type of data emitted
    + */
    +public class FlinkKinesisConsumer<T> extends RichParallelSourceFunction<T>
    +   implements CheckpointListener, 
CheckpointedAsynchronously<HashMap<KinesisStreamShard, String>>, 
ResultTypeQueryable<T> {
    --- End diff --
    
    I don't think we need to implement the `CheckpointListener` currently 
because we are not committing the offsets anywhere.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastruct...@apache.org or file a JIRA ticket
with INFRA.
---

Reply via email to