提交 844e0b2a 编写于 作者: S Stephan Ewen

[FLINK-2386] [kafka connector] Move Kafka connector classes to...

[FLINK-2386] [kafka connector] Move Kafka connector classes to 'org.apache.flink.streaming.connectors.kafka'
上级 76fcaca8
......@@ -15,7 +15,7 @@
* limitations under the License.
*/
package org.apache.flink.streaming.connectors;
package org.apache.flink.streaming.connectors.kafka;
import kafka.cluster.Broker;
import kafka.common.ErrorMapping;
......@@ -31,10 +31,10 @@ import org.apache.flink.configuration.Configuration;
import org.apache.flink.streaming.api.checkpoint.CheckpointNotifier;
import org.apache.flink.streaming.api.checkpoint.CheckpointedAsynchronously;
import org.apache.flink.streaming.api.functions.source.RichParallelSourceFunction;
import org.apache.flink.streaming.connectors.internals.Fetcher;
import org.apache.flink.streaming.connectors.internals.LegacyFetcher;
import org.apache.flink.streaming.connectors.internals.OffsetHandler;
import org.apache.flink.streaming.connectors.internals.ZookeeperOffsetHandler;
import org.apache.flink.streaming.connectors.kafka.internals.Fetcher;
import org.apache.flink.streaming.connectors.kafka.internals.LegacyFetcher;
import org.apache.flink.streaming.connectors.kafka.internals.OffsetHandler;
import org.apache.flink.streaming.connectors.kafka.internals.ZookeeperOffsetHandler;
import org.apache.flink.streaming.util.serialization.DeserializationSchema;
import org.apache.flink.util.NetUtils;
......
......@@ -15,7 +15,7 @@
* limitations under the License.
*/
package org.apache.flink.streaming.connectors;
package org.apache.flink.streaming.connectors.kafka;
import org.apache.flink.streaming.util.serialization.DeserializationSchema;
......
......@@ -15,7 +15,7 @@
* limitations under the License.
*/
package org.apache.flink.streaming.connectors;
package org.apache.flink.streaming.connectors.kafka;
import org.apache.flink.streaming.util.serialization.DeserializationSchema;
......
......@@ -15,7 +15,7 @@
* limitations under the License.
*/
package org.apache.flink.streaming.connectors;
package org.apache.flink.streaming.connectors.kafka;
import com.google.common.base.Preconditions;
import kafka.javaapi.producer.Producer;
......@@ -25,6 +25,7 @@ import kafka.serializer.DefaultEncoder;
import org.apache.flink.api.java.ClosureCleaner;
import org.apache.flink.configuration.Configuration;
import org.apache.flink.streaming.api.functions.sink.RichSinkFunction;
import org.apache.flink.streaming.connectors.kafka.internals.PartitionerWrapper;
import org.apache.flink.streaming.util.serialization.SerializationSchema;
import org.apache.flink.util.NetUtils;
import org.slf4j.Logger;
......
......@@ -14,7 +14,7 @@
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.flink.streaming.connectors;
package org.apache.flink.streaming.connectors.kafka;
import kafka.producer.Partitioner;
......
......@@ -22,11 +22,11 @@ import org.apache.flink.streaming.util.serialization.SerializationSchema;
/**
* Sink that emits its inputs to a Kafka topic.
*
* The KafkaSink has been relocated to org.apache.flink.streaming.connectors.KafkaSink.
* The KafkaSink has been relocated to org.apache.flink.streaming.connectors.kafka.KafkaSink.
* This class will be removed in future releases of Flink.
*/
@Deprecated
public class KafkaSink<IN> extends org.apache.flink.streaming.connectors.KafkaSink<IN> {
public class KafkaSink<IN> extends org.apache.flink.streaming.connectors.kafka.KafkaSink<IN> {
public KafkaSink(String brokerList, String topicId, SerializationSchema<IN, byte[]> serializationSchema) {
super(brokerList, topicId, serializationSchema);
}
......
......@@ -18,7 +18,7 @@
package org.apache.flink.streaming.connectors.kafka.api.persistent;
import kafka.consumer.ConsumerConfig;
import org.apache.flink.streaming.connectors.FlinkKafkaConsumer;
import org.apache.flink.streaming.connectors.kafka.FlinkKafkaConsumer;
import org.apache.flink.streaming.util.serialization.DeserializationSchema;
......
......@@ -15,7 +15,7 @@
* limitations under the License.
*/
package org.apache.flink.streaming.connectors.internals;
package org.apache.flink.streaming.connectors.kafka.internals;
import org.apache.flink.streaming.api.functions.source.SourceFunction;
import org.apache.flink.streaming.util.serialization.DeserializationSchema;
......
......@@ -15,7 +15,7 @@
* limitations under the License.
*/
package org.apache.flink.streaming.connectors.internals;
package org.apache.flink.streaming.connectors.kafka.internals;
import kafka.api.FetchRequestBuilder;
import kafka.api.OffsetRequest;
......@@ -29,7 +29,7 @@ import kafka.javaapi.message.ByteBufferMessageSet;
import kafka.message.MessageAndOffset;
import org.apache.flink.streaming.api.functions.source.SourceFunction;
import org.apache.flink.streaming.connectors.FlinkKafkaConsumer;
import org.apache.flink.streaming.connectors.kafka.FlinkKafkaConsumer;
import org.apache.flink.streaming.util.serialization.DeserializationSchema;
import org.apache.flink.util.StringUtils;
......
......@@ -14,7 +14,7 @@
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.flink.streaming.connectors;
package org.apache.flink.streaming.connectors.kafka.internals;
import kafka.producer.Partitioner;
import kafka.utils.VerifiableProperties;
......
......@@ -16,14 +16,14 @@
* limitations under the License.
*/
package org.apache.flink.streaming.connectors.internals;
package org.apache.flink.streaming.connectors.kafka.internals;
import kafka.common.TopicAndPartition;
import kafka.utils.ZKGroupTopicDirs;
import kafka.utils.ZkUtils;
import org.I0Itec.zkclient.ZkClient;
import org.apache.flink.streaming.connectors.FlinkKafkaConsumer;
import org.apache.flink.streaming.connectors.kafka.FlinkKafkaConsumer;
import org.apache.kafka.clients.consumer.ConsumerConfig;
import org.apache.kafka.common.TopicPartition;
import org.apache.zookeeper.data.Stat;
......
......@@ -16,9 +16,10 @@
* limitations under the License.
*/
package org.apache.flink.streaming.connectors;
package org.apache.flink.streaming.connectors.kafka;
import org.apache.flink.streaming.connectors.kafka.FlinkKafkaConsumer;
import org.apache.kafka.common.TopicPartition;
import org.junit.Test;
......
......@@ -16,10 +16,11 @@
* limitations under the License.
*/
package org.apache.flink.streaming.connectors;
package org.apache.flink.streaming.connectors.kafka;
import org.apache.commons.collections.map.LinkedMap;
import org.apache.flink.streaming.connectors.kafka.FlinkKafkaConsumer;
import org.apache.flink.streaming.util.serialization.JavaDefaultStringSchema;
import org.apache.kafka.clients.consumer.ConsumerConfig;
import org.junit.Ignore;
......
......@@ -16,7 +16,7 @@
* limitations under the License.
*/
package org.apache.flink.streaming.connectors;
package org.apache.flink.streaming.connectors.kafka;
import kafka.admin.AdminUtils;
......@@ -51,17 +51,17 @@ import org.apache.flink.streaming.api.functions.sink.RichSinkFunction;
import org.apache.flink.streaming.api.functions.sink.SinkFunction;
import org.apache.flink.streaming.api.functions.source.RichParallelSourceFunction;
import org.apache.flink.streaming.api.functions.source.RichSourceFunction;
import org.apache.flink.streaming.connectors.internals.ZookeeperOffsetHandler;
import org.apache.flink.streaming.connectors.testutils.DataGenerators;
import org.apache.flink.streaming.connectors.testutils.DiscardingSink;
import org.apache.flink.streaming.connectors.testutils.FailingIdentityMapper;
import org.apache.flink.streaming.connectors.testutils.JobManagerCommunicationUtils;
import org.apache.flink.streaming.connectors.testutils.MockRuntimeContext;
import org.apache.flink.streaming.connectors.testutils.PartitionValidatingMapper;
import org.apache.flink.streaming.connectors.testutils.SuccessException;
import org.apache.flink.streaming.connectors.testutils.ThrottledMapper;
import org.apache.flink.streaming.connectors.testutils.Tuple2Partitioner;
import org.apache.flink.streaming.connectors.testutils.ValidatingExactlyOnceSink;
import org.apache.flink.streaming.connectors.kafka.internals.ZookeeperOffsetHandler;
import org.apache.flink.streaming.connectors.kafka.testutils.DataGenerators;
import org.apache.flink.streaming.connectors.kafka.testutils.DiscardingSink;
import org.apache.flink.streaming.connectors.kafka.testutils.FailingIdentityMapper;
import org.apache.flink.streaming.connectors.kafka.testutils.JobManagerCommunicationUtils;
import org.apache.flink.streaming.connectors.kafka.testutils.MockRuntimeContext;
import org.apache.flink.streaming.connectors.kafka.testutils.PartitionValidatingMapper;
import org.apache.flink.streaming.connectors.kafka.testutils.SuccessException;
import org.apache.flink.streaming.connectors.kafka.testutils.ThrottledMapper;
import org.apache.flink.streaming.connectors.kafka.testutils.Tuple2Partitioner;
import org.apache.flink.streaming.connectors.kafka.testutils.ValidatingExactlyOnceSink;
import org.apache.flink.streaming.util.serialization.DeserializationSchema;
import org.apache.flink.streaming.util.serialization.JavaDefaultStringSchema;
import org.apache.flink.streaming.util.serialization.TypeInformationSerializationSchema;
......
......@@ -15,7 +15,7 @@
* limitations under the License.
*/
package org.apache.flink.streaming.connectors;
package org.apache.flink.streaming.connectors.kafka;
import org.apache.flink.streaming.util.serialization.DeserializationSchema;
......
......@@ -16,7 +16,7 @@
* limitations under the License.
*/
package org.apache.flink.streaming.connectors;
package org.apache.flink.streaming.connectors.kafka;
import org.apache.flink.api.common.functions.RichMapFunction;
import org.apache.flink.api.common.typeinfo.TypeInformation;
......@@ -26,7 +26,7 @@ import org.apache.flink.streaming.api.datastream.DataStream;
import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment;
import org.apache.flink.streaming.api.functions.sink.SinkFunction;
import org.apache.flink.streaming.api.functions.source.SourceFunction;
import org.apache.flink.streaming.connectors.testutils.SuccessException;
import org.apache.flink.streaming.connectors.kafka.testutils.SuccessException;
import org.apache.flink.streaming.util.serialization.TypeInformationSerializationSchema;
import org.junit.Test;
......
......@@ -15,7 +15,7 @@
* limitations under the License.
*/
package org.apache.flink.streaming.connectors;
package org.apache.flink.streaming.connectors.kafka;
import kafka.admin.AdminUtils;
import kafka.consumer.ConsumerConfig;
......@@ -34,8 +34,8 @@ import org.apache.flink.runtime.StreamingMode;
import org.apache.flink.runtime.client.JobExecutionException;
import org.apache.flink.runtime.net.NetUtils;
import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment;
import org.apache.flink.streaming.connectors.internals.ZooKeeperStringSerializer;
import org.apache.flink.streaming.connectors.testutils.SuccessException;
import org.apache.flink.streaming.connectors.kafka.internals.ZooKeeperStringSerializer;
import org.apache.flink.streaming.connectors.kafka.testutils.SuccessException;
import org.apache.flink.test.util.ForkableFlinkMiniCluster;
import org.apache.kafka.common.PartitionInfo;
......
......@@ -16,13 +16,14 @@
* limitations under the License.
*/
package org.apache.flink.streaming.connectors.internals;
package org.apache.flink.streaming.connectors.kafka.internals;
import kafka.admin.AdminUtils;
import org.I0Itec.zkclient.ZkClient;
import org.apache.flink.streaming.connectors.KafkaTestBase;
import org.apache.flink.streaming.connectors.kafka.KafkaTestBase;
import org.apache.flink.streaming.connectors.kafka.internals.ZookeeperOffsetHandler;
import org.junit.Test;
import java.util.Properties;
......
......@@ -16,7 +16,7 @@
* limitations under the License.
*/
package org.apache.flink.streaming.connectors.testutils;
package org.apache.flink.streaming.connectors.kafka.testutils;
import org.apache.flink.api.common.typeinfo.BasicTypeInfo;
import org.apache.flink.api.common.typeinfo.TypeInformation;
......@@ -26,8 +26,8 @@ import org.apache.flink.configuration.Configuration;
import org.apache.flink.streaming.api.datastream.DataStream;
import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment;
import org.apache.flink.streaming.api.functions.source.RichParallelSourceFunction;
import org.apache.flink.streaming.connectors.KafkaSink;
import org.apache.flink.streaming.connectors.SerializableKafkaPartitioner;
import org.apache.flink.streaming.connectors.kafka.KafkaSink;
import org.apache.flink.streaming.connectors.kafka.SerializableKafkaPartitioner;
import org.apache.flink.streaming.util.serialization.JavaDefaultStringSchema;
import org.apache.flink.streaming.util.serialization.TypeInformationSerializationSchema;
......
......@@ -16,7 +16,7 @@
* limitations under the License.
*/
package org.apache.flink.streaming.connectors.testutils;
package org.apache.flink.streaming.connectors.kafka.testutils;
import org.apache.flink.streaming.api.functions.sink.SinkFunction;
......
......@@ -16,7 +16,7 @@
* limitations under the License.
*/
package org.apache.flink.streaming.connectors.testutils;
package org.apache.flink.streaming.connectors.kafka.testutils;
import org.apache.flink.api.common.functions.RichMapFunction;
import org.apache.flink.configuration.Configuration;
......
......@@ -16,7 +16,7 @@
* limitations under the License.
*/
package org.apache.flink.streaming.connectors.testutils;
package org.apache.flink.streaming.connectors.kafka.testutils;
import org.apache.flink.api.common.JobID;
import org.apache.flink.runtime.client.JobStatusMessage;
......
......@@ -16,7 +16,7 @@
* limitations under the License.
*/
package org.apache.flink.streaming.connectors.testutils;
package org.apache.flink.streaming.connectors.kafka.testutils;
import org.apache.flink.api.common.ExecutionConfig;
import org.apache.flink.api.common.accumulators.Accumulator;
......
......@@ -16,7 +16,7 @@
* limitations under the License.
*/
package org.apache.flink.streaming.connectors.testutils;
package org.apache.flink.streaming.connectors.kafka.testutils;
/**
* Exception that is thrown to terminate a program and indicate success.
......
......@@ -16,7 +16,7 @@
* limitations under the License.
*/
package org.apache.flink.streaming.connectors.testutils;
package org.apache.flink.streaming.connectors.kafka.testutils;
import org.apache.flink.api.common.functions.MapFunction;
......
......@@ -16,10 +16,10 @@
* limitations under the License.
*/
package org.apache.flink.streaming.connectors.testutils;
package org.apache.flink.streaming.connectors.kafka.testutils;
import org.apache.flink.api.java.tuple.Tuple2;
import org.apache.flink.streaming.connectors.SerializableKafkaPartitioner;
import org.apache.flink.streaming.connectors.kafka.SerializableKafkaPartitioner;
/**
* Special partitioner that uses the first field of a 2-tuple as the partition,
......
......@@ -16,7 +16,7 @@
* limitations under the License.
*/
package org.apache.flink.streaming.connectors.testutils;
package org.apache.flink.streaming.connectors.kafka.testutils;
import org.apache.flink.api.java.tuple.Tuple2;
import org.apache.flink.streaming.api.checkpoint.Checkpointed;
......
Markdown is supported
0% .
You are about to add 0 people to the discussion. Proceed with caution.
先完成此消息的编辑!
想要评论请 注册