[2/2] flink git commit: [FLINK-4738] [TaskManager] Port TaskManager logic to new Flip-6 TaskManager

2016-10-11 Thread trohrmann
[FLINK-4738] [TaskManager] Port TaskManager logic to new Flip-6 TaskManager

The ported logic contains the task lifecycle management methods, JobManager 
association and
setup of TaskManager components.

Introduce Rpc implementations for TaskManager components

Implement metrics setup

Move more TaskManager components out of the constructor to make TaskExecutor 
more testable

Add RpcMethod annotation to TaskExecutor#confirmCheckpoint

This closes #2594.


Project: http://git-wip-us.apache.org/repos/asf/flink/repo
Commit: http://git-wip-us.apache.org/repos/asf/flink/commit/8c656d92
Tree: http://git-wip-us.apache.org/repos/asf/flink/tree/8c656d92
Diff: http://git-wip-us.apache.org/repos/asf/flink/diff/8c656d92

Branch: refs/heads/flip-6
Commit: 8c656d9252fd01ea82c75ff1ea830b110f68f8a9
Parents: f863d16
Author: Till Rohrmann 
Authored: Wed Sep 28 14:39:51 2016 +0200
Committer: Till Rohrmann 
Committed: Tue Oct 11 17:22:34 2016 +0200

--
 .../CheckpointCoordinatorGateway.java   |  43 ++
 .../deployment/TaskDeploymentDescriptor.java|   9 +
 .../runtime/executiongraph/PartitionInfo.java   |  47 ++
 .../flink/runtime/filecache/FileCache.java  |  17 +-
 .../jobgraph/tasks/InputSplitProvider.java  |   3 +-
 .../tasks/InputSplitProviderException.java  |  36 ++
 .../jobmaster/ExecutionGraphException.java  |  41 ++
 .../runtime/jobmaster/JobManagerException.java  |  39 ++
 .../flink/runtime/jobmaster/JobMaster.java  |  50 +-
 .../runtime/jobmaster/JobMasterGateway.java |  30 +-
 .../jobmaster/MiniClusterJobDispatcher.java |   2 +-
 .../runtime/jobmaster/SerializedInputSplit.java |  39 ++
 .../jobmaster/message/NextInputSplit.java   |  39 --
 .../flink/runtime/operators/DataSourceTask.java |  12 +-
 .../runtime/query/KvStateRegistryGateway.java   |  57 ++
 .../taskexecutor/JobManagerConnection.java  |  91 +++
 .../runtime/taskexecutor/TaskExecutor.java  | 627 +--
 .../taskexecutor/TaskExecutorGateway.java   |  80 ++-
 .../taskexecutor/TaskManagerConfiguration.java  |   3 +-
 .../runtime/taskexecutor/TaskManagerRunner.java |   3 +
 .../taskexecutor/TaskManagerServices.java   |  51 +-
 .../flink/runtime/taskexecutor/TaskSlot.java|  73 +++
 .../runtime/taskexecutor/TaskSlotMapping.java   |  44 ++
 .../exceptions/CheckpointException.java |  41 ++
 .../exceptions/PartitionException.java  |  41 ++
 .../taskexecutor/exceptions/TaskException.java  |  41 ++
 .../exceptions/TaskManagerException.java|  41 ++
 .../exceptions/TaskSubmissionException.java |  41 ++
 .../rpc/RpcCheckpointResponder.java |  71 +++
 .../taskexecutor/rpc/RpcInputSplitProvider.java |  73 +++
 .../rpc/RpcKvStateRegistryListener.java |  73 +++
 .../rpc/RpcPartitionStateChecker.java   |  48 ++
 .../RpcResultPartitionConsumableNotifier.java   |  67 ++
 .../utils/TaskExecutorMetricsInitializer.java   | 257 
 .../ActorGatewayTaskManagerActions.java |  59 ++
 .../ActorGatewayTaskManagerConnection.java  |  59 --
 .../apache/flink/runtime/taskmanager/Task.java  |  23 +-
 .../runtime/taskmanager/TaskExecutionState.java |   4 +-
 .../taskmanager/TaskInputSplitProvider.java |  49 +-
 .../runtime/taskmanager/TaskManagerActions.java |  57 ++
 .../taskmanager/TaskManagerConnection.java  |  57 --
 .../flink/runtime/taskmanager/TaskManager.scala | 169 +
 .../FileCacheDeleteValidationTest.java  |   4 +-
 .../jobmaster/JobManagerRunnerMockTest.java |   1 -
 .../runtime/taskexecutor/TaskExecutorTest.java  |  24 +-
 .../runtime/taskmanager/TaskAsyncCallTest.java  |   2 +-
 .../taskmanager/TaskInputSplitProviderTest.java |   3 +-
 .../flink/runtime/taskmanager/TaskStopTest.java |   4 +-
 .../flink/runtime/taskmanager/TaskTest.java |   4 +-
 .../source/InputFormatSourceFunction.java   |   8 +-
 .../tasks/InterruptSensitiveRestoreTest.java|  11 +-
 .../streaming/runtime/tasks/StreamTaskTest.java |   4 +-
 52 files changed, 2340 insertions(+), 432 deletions(-)
--


http://git-wip-us.apache.org/repos/asf/flink/blob/8c656d92/flink-runtime/src/main/java/org/apache/flink/runtime/checkpoint/CheckpointCoordinatorGateway.java
--
diff --git 
a/flink-runtime/src/main/java/org/apache/flink/runtime/checkpoint/CheckpointCoordinatorGateway.java
 
b/flink-runtime/src/main/java/org/apache/flink/runtime/checkpoint/CheckpointCoordinatorGateway.java
new file mode 100644
index 000..e448ebc
--- /dev/null
+++ 
b/flink-runtime/src/main/java/org/apache/flink/runtime/checkpoint/CheckpointCoordinatorGateway.java
@@ -0,0 +1,43 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * 

[1/2] flink git commit: [FLINK-4738] [TaskManager] Port TaskManager logic to new Flip-6 TaskManager

2016-10-11 Thread trohrmann
Repository: flink
Updated Branches:
  refs/heads/flip-6 f863d1686 -> 8c656d925


http://git-wip-us.apache.org/repos/asf/flink/blob/8c656d92/flink-runtime/src/main/java/org/apache/flink/runtime/taskexecutor/exceptions/CheckpointException.java
--
diff --git 
a/flink-runtime/src/main/java/org/apache/flink/runtime/taskexecutor/exceptions/CheckpointException.java
 
b/flink-runtime/src/main/java/org/apache/flink/runtime/taskexecutor/exceptions/CheckpointException.java
new file mode 100644
index 000..80f2aa0
--- /dev/null
+++ 
b/flink-runtime/src/main/java/org/apache/flink/runtime/taskexecutor/exceptions/CheckpointException.java
@@ -0,0 +1,41 @@
+/*
+ * 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.runtime.taskexecutor.exceptions;
+
+import org.apache.flink.runtime.taskexecutor.TaskExecutor;
+
+/**
+ * Exception indicating a problem with checkpointing on the {@link 
TaskExecutor} side.
+ */
+public class CheckpointException extends TaskManagerException {
+
+   private static final long serialVersionUID = 3366394086880327955L;
+
+   public CheckpointException(String message) {
+   super(message);
+   }
+
+   public CheckpointException(String message, Throwable cause) {
+   super(message, cause);
+   }
+
+   public CheckpointException(Throwable cause) {
+   super(cause);
+   }
+}

http://git-wip-us.apache.org/repos/asf/flink/blob/8c656d92/flink-runtime/src/main/java/org/apache/flink/runtime/taskexecutor/exceptions/PartitionException.java
--
diff --git 
a/flink-runtime/src/main/java/org/apache/flink/runtime/taskexecutor/exceptions/PartitionException.java
 
b/flink-runtime/src/main/java/org/apache/flink/runtime/taskexecutor/exceptions/PartitionException.java
new file mode 100644
index 000..eecd0ae
--- /dev/null
+++ 
b/flink-runtime/src/main/java/org/apache/flink/runtime/taskexecutor/exceptions/PartitionException.java
@@ -0,0 +1,41 @@
+/*
+ * 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.runtime.taskexecutor.exceptions;
+
+import org.apache.flink.runtime.taskexecutor.TaskExecutor;
+
+/**
+ * Exception indicating a problem with the result partitions on the {@link 
TaskExecutor} side.
+ */
+public class PartitionException extends TaskManagerException {
+
+   private static final long serialVersionUID = 6248696963418276618L;
+
+   public PartitionException(String message) {
+   super(message);
+   }
+
+   public PartitionException(String message, Throwable cause) {
+   super(message, cause);
+   }
+
+   public PartitionException(Throwable cause) {
+   super(cause);
+   }
+}

http://git-wip-us.apache.org/repos/asf/flink/blob/8c656d92/flink-runtime/src/main/java/org/apache/flink/runtime/taskexecutor/exceptions/TaskException.java
--
diff --git 
a/flink-runtime/src/main/java/org/apache/flink/runtime/taskexecutor/exceptions/TaskException.java
 
b/flink-runtime/src/main/java/org/apache/flink/runtime/taskexecutor/exceptions/TaskException.java
new file mode 100644
index 000..a4a89c2
--- /dev/null
+++ 
b/flink-runtime/src/main/java/org/apache/flink/runtime/taskexecutor/exceptions/TaskException.java
@@ -0,0 +1,41 @@
+/*
+ * 

flink git commit: [FLINK-4694] [rpc] Add termination futures to RpcEndpoint and RpcService

2016-10-11 Thread trohrmann
Repository: flink
Updated Branches:
  refs/heads/flip-6 0518af03a -> f863d1686


[FLINK-4694] [rpc] Add termination futures to RpcEndpoint and RpcService

The termination futures can be used to wait for the termination of the 
respective component.

This closes #2558.


Project: http://git-wip-us.apache.org/repos/asf/flink/repo
Commit: http://git-wip-us.apache.org/repos/asf/flink/commit/f863d168
Tree: http://git-wip-us.apache.org/repos/asf/flink/tree/f863d168
Diff: http://git-wip-us.apache.org/repos/asf/flink/diff/f863d168

Branch: refs/heads/flip-6
Commit: f863d16860be24ab6932282db0722eadc51b64a6
Parents: 0518af0
Author: Till Rohrmann 
Authored: Tue Sep 27 18:17:42 2016 +0200
Committer: Till Rohrmann 
Committed: Tue Oct 11 15:48:37 2016 +0200

--
 .../concurrent/impl/FlinkCompletableFuture.java | 11 +++---
 .../apache/flink/runtime/rpc/RpcEndpoint.java   |  9 +
 .../apache/flink/runtime/rpc/RpcService.java|  7 
 .../apache/flink/runtime/rpc/SelfGateway.java   | 34 ++
 .../runtime/rpc/akka/AkkaInvocationHandler.java | 22 ++--
 .../flink/runtime/rpc/akka/AkkaRpcActor.java| 17 -
 .../flink/runtime/rpc/akka/AkkaRpcService.java  | 32 +++--
 .../runtime/rpc/TestingSerialRpcService.java| 10 +-
 .../runtime/rpc/akka/AkkaRpcActorTest.java  | 36 
 .../runtime/rpc/akka/AkkaRpcServiceTest.java| 29 
 10 files changed, 193 insertions(+), 14 deletions(-)
--


http://git-wip-us.apache.org/repos/asf/flink/blob/f863d168/flink-runtime/src/main/java/org/apache/flink/runtime/concurrent/impl/FlinkCompletableFuture.java
--
diff --git 
a/flink-runtime/src/main/java/org/apache/flink/runtime/concurrent/impl/FlinkCompletableFuture.java
 
b/flink-runtime/src/main/java/org/apache/flink/runtime/concurrent/impl/FlinkCompletableFuture.java
index e648a71..14686d7 100644
--- 
a/flink-runtime/src/main/java/org/apache/flink/runtime/concurrent/impl/FlinkCompletableFuture.java
+++ 
b/flink-runtime/src/main/java/org/apache/flink/runtime/concurrent/impl/FlinkCompletableFuture.java
@@ -20,7 +20,6 @@ package org.apache.flink.runtime.concurrent.impl;
 
 import akka.dispatch.Futures;
 import org.apache.flink.runtime.concurrent.CompletableFuture;
-import org.apache.flink.util.Preconditions;
 import scala.concurrent.Promise;
 import scala.concurrent.Promise$;
 
@@ -52,8 +51,6 @@ public class FlinkCompletableFuture extends FlinkFuture 
implements Complet
 
@Override
public boolean complete(T value) {
-   Preconditions.checkNotNull(value);
-
try {
promise.success(value);
 
@@ -65,10 +62,12 @@ public class FlinkCompletableFuture extends 
FlinkFuture implements Complet
 
@Override
public boolean completeExceptionally(Throwable t) {
-   Preconditions.checkNotNull(t);
-
try {
-   promise.failure(t);
+   if (t == null) {
+   promise.failure(new 
NullPointerException("Throwable was null."));
+   } else {
+   promise.failure(t);
+   }
 
return true;
} catch (IllegalStateException e) {

http://git-wip-us.apache.org/repos/asf/flink/blob/f863d168/flink-runtime/src/main/java/org/apache/flink/runtime/rpc/RpcEndpoint.java
--
diff --git 
a/flink-runtime/src/main/java/org/apache/flink/runtime/rpc/RpcEndpoint.java 
b/flink-runtime/src/main/java/org/apache/flink/runtime/rpc/RpcEndpoint.java
index 79961f7..f93a2e2 100644
--- a/flink-runtime/src/main/java/org/apache/flink/runtime/rpc/RpcEndpoint.java
+++ b/flink-runtime/src/main/java/org/apache/flink/runtime/rpc/RpcEndpoint.java
@@ -173,6 +173,15 @@ public abstract class RpcEndpoint {
return rpcService;
}
 
+   /**
+* Return a future which is completed when the rpc endpoint has been 
terminated.
+*
+* @return Future which is completed when the rpc endpoint has been 
terminated.
+*/
+   public Future getTerminationFuture() {
+   return ((SelfGateway)self).getTerminationFuture();
+   }
+
// 

//  Asynchronous executions
// 


http://git-wip-us.apache.org/repos/asf/flink/blob/f863d168/flink-runtime/src/main/java/org/apache/flink/runtime/rpc/RpcService.java
--
diff --git 

buildbot success in on flink-docs-release-1.1

2016-10-11 Thread buildbot
The Buildbot has detected a restored build on builder flink-docs-release-1.1 
while building . Full details are available at:
https://ci.apache.org/builders/flink-docs-release-1.1/builds/75

Buildbot URL: https://ci.apache.org/

Buildslave for this Build: bb_slave1_ubuntu

Build Reason: The Nightly scheduler named 'flink-nightly-docs-release-1.1' 
triggered this build
Build Source Stamp: [branch release-1.1] HEAD
Blamelist: 

Build succeeded!

Sincerely,
 -The Buildbot





buildbot failure in on flink-docs-release-0.9

2016-10-11 Thread buildbot
The Buildbot has detected a new failure on builder flink-docs-release-0.9 while 
building . Full details are available at:
https://ci.apache.org/builders/flink-docs-release-0.9/builds/473

Buildbot URL: https://ci.apache.org/

Buildslave for this Build: bb_slave2_ubuntu

Build Reason: The Nightly scheduler named 'flink-nightly-docs-release-0.9' 
triggered this build
Build Source Stamp: [branch release-0.9] HEAD
Blamelist: 

BUILD FAILED: failed Flink docs

Sincerely,
 -The Buildbot





[1/3] flink git commit: [FLINK-4035] Refactor the Kafka 0.10 connector to be based upon the 0.9 connector

2016-10-11 Thread rmetzger
Repository: flink
Updated Branches:
  refs/heads/master a079259f3 -> 6731ec1e4


http://git-wip-us.apache.org/repos/asf/flink/blob/6731ec1e/flink-streaming-connectors/flink-connector-kafka-0.9/src/test/java/org/apache/flink/streaming/connectors/kafka/KafkaTestEnvironmentImpl.java
--
diff --git 
a/flink-streaming-connectors/flink-connector-kafka-0.9/src/test/java/org/apache/flink/streaming/connectors/kafka/KafkaTestEnvironmentImpl.java
 
b/flink-streaming-connectors/flink-connector-kafka-0.9/src/test/java/org/apache/flink/streaming/connectors/kafka/KafkaTestEnvironmentImpl.java
index 213ba4a..9d8fa9a 100644
--- 
a/flink-streaming-connectors/flink-connector-kafka-0.9/src/test/java/org/apache/flink/streaming/connectors/kafka/KafkaTestEnvironmentImpl.java
+++ 
b/flink-streaming-connectors/flink-connector-kafka-0.9/src/test/java/org/apache/flink/streaming/connectors/kafka/KafkaTestEnvironmentImpl.java
@@ -29,6 +29,8 @@ import kafka.utils.ZkUtils;
 import org.I0Itec.zkclient.ZkClient;
 import org.apache.commons.io.FileUtils;
 import org.apache.curator.test.TestingServer;
+import org.apache.flink.streaming.api.datastream.DataStream;
+import org.apache.flink.streaming.api.datastream.DataStreamSink;
 import 
org.apache.flink.streaming.connectors.kafka.testutils.ZooKeeperStringSerializer;
 import 
org.apache.flink.streaming.connectors.kafka.partitioner.KafkaPartitioner;
 import 
org.apache.flink.streaming.util.serialization.KeyedDeserializationSchema;
@@ -94,10 +96,10 @@ public class KafkaTestEnvironmentImpl extends 
KafkaTestEnvironment {
}
 
@Override
-   public  FlinkKafkaProducerBase getProducer(String topic, 
KeyedSerializationSchema serSchema, Properties props, KafkaPartitioner 
partitioner) {
+   public  DataStreamSink produceIntoKafka(DataStream stream, 
String topic, KeyedSerializationSchema serSchema, Properties props, 
KafkaPartitioner partitioner) {
FlinkKafkaProducer09 prod = new 
FlinkKafkaProducer09<>(topic, serSchema, props, partitioner);
prod.setFlushOnCheckpoint(true);
-   return prod;
+   return stream.addSink(prod);
}
 
@Override

http://git-wip-us.apache.org/repos/asf/flink/blob/6731ec1e/flink-streaming-connectors/flink-connector-kafka-base/src/main/java/org/apache/flink/streaming/connectors/kafka/FlinkKafkaConsumerBase.java
--
diff --git 
a/flink-streaming-connectors/flink-connector-kafka-base/src/main/java/org/apache/flink/streaming/connectors/kafka/FlinkKafkaConsumerBase.java
 
b/flink-streaming-connectors/flink-connector-kafka-base/src/main/java/org/apache/flink/streaming/connectors/kafka/FlinkKafkaConsumerBase.java
index 5b18c75..05028e6 100644
--- 
a/flink-streaming-connectors/flink-connector-kafka-base/src/main/java/org/apache/flink/streaming/connectors/kafka/FlinkKafkaConsumerBase.java
+++ 
b/flink-streaming-connectors/flink-connector-kafka-base/src/main/java/org/apache/flink/streaming/connectors/kafka/FlinkKafkaConsumerBase.java
@@ -21,6 +21,7 @@ import org.apache.commons.collections.map.LinkedMap;
 import org.apache.flink.api.common.state.ListState;
 import org.apache.flink.api.common.typeinfo.TypeInformation;
 import org.apache.flink.api.java.tuple.Tuple2;
+import org.apache.flink.api.java.ClosureCleaner;
 import org.apache.flink.api.java.typeutils.ResultTypeQueryable;
 import org.apache.flink.configuration.Configuration;
 import org.apache.flink.runtime.state.CheckpointListener;
@@ -36,6 +37,7 @@ import 
org.apache.flink.streaming.connectors.kafka.internals.KafkaTopicPartition
 import 
org.apache.flink.streaming.connectors.kafka.internals.KafkaTopicPartitionState;
 import 
org.apache.flink.streaming.util.serialization.KeyedDeserializationSchema;
 import org.apache.flink.util.SerializedValue;
+
 import org.slf4j.Logger;
 import org.slf4j.LoggerFactory;
 
@@ -169,6 +171,7 @@ public abstract class FlinkKafkaConsumerBase extends 
RichParallelSourceFuncti
throw new IllegalStateException("A periodic watermark 
emitter has already been set.");
}
try {
+   ClosureCleaner.clean(assigner, true);
this.punctuatedWatermarkAssigner = new 
SerializedValue<>(assigner);
return this;
} catch (Exception e) {
@@ -203,6 +206,7 @@ public abstract class FlinkKafkaConsumerBase extends 
RichParallelSourceFuncti
throw new IllegalStateException("A punctuated watermark 
emitter has already been set.");
}
try {
+   ClosureCleaner.clean(assigner, true);
this.periodicWatermarkAssigner = new 
SerializedValue<>(assigner);
return this;
} catch (Exception e) {


[2/3] flink git commit: [FLINK-4035] Refactor the Kafka 0.10 connector to be based upon the 0.9 connector

2016-10-11 Thread rmetzger
[FLINK-4035] Refactor the Kafka 0.10 connector to be based upon the 0.9 
connector

Add a test case for Kafka's new timestamp functionality and update the 
documentation.

This closes #2369


Project: http://git-wip-us.apache.org/repos/asf/flink/repo
Commit: http://git-wip-us.apache.org/repos/asf/flink/commit/6731ec1e
Tree: http://git-wip-us.apache.org/repos/asf/flink/tree/6731ec1e
Diff: http://git-wip-us.apache.org/repos/asf/flink/diff/6731ec1e

Branch: refs/heads/master
Commit: 6731ec1e48d0a0092dd2330adda73bcf37fda8d7
Parents: 63859c6
Author: Robert Metzger 
Authored: Tue Aug 9 16:38:21 2016 +0200
Committer: Robert Metzger 
Committed: Tue Oct 11 10:04:25 2016 +0200

--
 docs/dev/connectors/kafka.md|  67 +++-
 docs/page/js/flink.js   |   3 +-
 .../flink-connector-kafka-0.10/pom.xml  |  50 +--
 .../connectors/kafka/FlinkKafkaConsumer010.java | 121 +--
 .../connectors/kafka/FlinkKafkaProducer010.java | 315 +--
 .../kafka/Kafka010JsonTableSource.java  |   2 +-
 .../connectors/kafka/Kafka010TableSource.java   |   2 +-
 .../kafka/internal/Kafka010Fetcher.java | 268 ++--
 .../connectors/kafka/Kafka010ITCase.java| 266 +++-
 .../connectors/kafka/KafkaProducerTest.java | 119 ---
 .../kafka/KafkaTestEnvironmentImpl.java |  80 -
 .../kafka/internals/SimpleConsumerThread.java   |   2 +-
 .../kafka/KafkaTestEnvironmentImpl.java |   7 +-
 .../connectors/kafka/FlinkKafkaConsumer09.java  |   4 +-
 .../connectors/kafka/FlinkKafkaProducer09.java  |   2 +-
 .../kafka/internal/Kafka09Fetcher.java  |  22 +-
 .../kafka/KafkaTestEnvironmentImpl.java |   6 +-
 .../kafka/FlinkKafkaConsumerBase.java   |   4 +
 .../kafka/FlinkKafkaProducerBase.java   |   4 +-
 .../kafka/internals/AbstractFetcher.java|  43 +--
 ...picPartitionStateWithPeriodicWatermarks.java |   4 +-
 ...cPartitionStateWithPunctuatedWatermarks.java |   4 +-
 .../connectors/kafka/KafkaConsumerTestBase.java | 201 ++--
 .../connectors/kafka/KafkaProducerTestBase.java |   5 +-
 .../kafka/KafkaShortRetentionTestBase.java  |   4 +-
 .../connectors/kafka/KafkaTestEnvironment.java  |   7 +-
 .../AbstractFetcherTimestampsTest.java  |  68 ++--
 .../kafka/testutils/DataGenerators.java |  87 ++---
 .../testutils/JobManagerCommunicationUtils.java |  21 +-
 29 files changed, 936 insertions(+), 852 deletions(-)
--


http://git-wip-us.apache.org/repos/asf/flink/blob/6731ec1e/docs/dev/connectors/kafka.md
--
diff --git a/docs/dev/connectors/kafka.md b/docs/dev/connectors/kafka.md
index d2221fa..9a360d4 100644
--- a/docs/dev/connectors/kafka.md
+++ b/docs/dev/connectors/kafka.md
@@ -46,14 +46,6 @@ For most users, the `FlinkKafkaConsumer08` (part of 
`flink-connector-kafka`) is
   
   
 
-flink-connector-kafka
-0.9.1, 0.10
-FlinkKafkaConsumer082
-FlinkKafkaProducer
-0.8.x
-Uses the https://cwiki.apache.org/confluence/display/KAFKA/0.8.0+SimpleConsumer+Example;>SimpleConsumer
 API of Kafka internally. Offsets are committed to ZK by Flink.
-
- 
 flink-connector-kafka-0.8{{ site.scala_version_suffix }}
 1.0.0
 FlinkKafkaConsumer08
@@ -61,7 +53,7 @@ For most users, the `FlinkKafkaConsumer08` (part of 
`flink-connector-kafka`) is
 0.8.x
 Uses the https://cwiki.apache.org/confluence/display/KAFKA/0.8.0+SimpleConsumer+Example;>SimpleConsumer
 API of Kafka internally. Offsets are committed to ZK by Flink.
 
- 
+
 flink-connector-kafka-0.9{{ site.scala_version_suffix }}
 1.0.0
 FlinkKafkaConsumer09
@@ -69,6 +61,14 @@ For most users, the `FlinkKafkaConsumer08` (part of 
`flink-connector-kafka`) is
 0.9.x
 Uses the new http://kafka.apache.org/documentation.html#newconsumerapi;>Consumer 
API Kafka.
 
+
+flink-connector-kafka-0.10{{ site.scala_version_suffix }}
+1.2.0
+FlinkKafkaConsumer010
+FlinkKafkaProducer010
+0.10.x
+This connector supports https://cwiki.apache.org/confluence/display/KAFKA/KIP-32+-+Add+timestamps+to+Kafka+message;>Kafka
 messages with timestamps both for producing and consuming.
+
   
 
 
@@ -87,7 +87,6 @@ Note that the streaming connectors are currently not part of 
the binary distribu
 ### Installing Apache Kafka
 
 * Follow the instructions from [Kafka's 
quickstart](https://kafka.apache.org/documentation.html#quickstart) to download 
the code and launch a server (launching a Zookeeper and a Kafka server is 
required every time before starting the application).
-* On 32 bit computers 

[3/3] flink git commit: [FLINK-4035] Add support for Kafka 0.10.x.

2016-10-11 Thread rmetzger
[FLINK-4035] Add support for Kafka 0.10.x.

This closes #2231


Project: http://git-wip-us.apache.org/repos/asf/flink/repo
Commit: http://git-wip-us.apache.org/repos/asf/flink/commit/63859c64
Tree: http://git-wip-us.apache.org/repos/asf/flink/tree/63859c64
Diff: http://git-wip-us.apache.org/repos/asf/flink/diff/63859c64

Branch: refs/heads/master
Commit: 63859c648a2a0ff024228f6e0687d837b8896322
Parents: a079259
Author: radekg 
Authored: Tue Jul 12 13:19:01 2016 -0400
Committer: Robert Metzger 
Committed: Tue Oct 11 10:04:25 2016 +0200

--
 .../flink-connector-kafka-0.10/pom.xml  | 179 ++
 .../connectors/kafka/FlinkKafkaConsumer010.java | 259 +++
 .../connectors/kafka/FlinkKafkaProducer010.java | 137 
 .../kafka/Kafka010JsonTableSource.java  |  71 
 .../connectors/kafka/Kafka010TableSource.java   |  75 +
 .../kafka/internal/Kafka010Fetcher.java | 312 +
 .../src/main/resources/log4j.properties |  29 ++
 .../connectors/kafka/Kafka010ITCase.java| 192 +++
 .../kafka/Kafka010ProducerITCase.java   |  33 ++
 .../connectors/kafka/KafkaProducerTest.java | 119 +++
 .../kafka/KafkaShortRetention010ITCase.java |  34 ++
 .../kafka/KafkaTestEnvironmentImpl.java | 331 +++
 .../src/test/resources/log4j-test.properties|  30 ++
 .../src/test/resources/logback-test.xml |  30 ++
 flink-streaming-connectors/pom.xml  |   1 +
 15 files changed, 1832 insertions(+)
--


http://git-wip-us.apache.org/repos/asf/flink/blob/63859c64/flink-streaming-connectors/flink-connector-kafka-0.10/pom.xml
--
diff --git a/flink-streaming-connectors/flink-connector-kafka-0.10/pom.xml 
b/flink-streaming-connectors/flink-connector-kafka-0.10/pom.xml
new file mode 100644
index 000..f2bcb11
--- /dev/null
+++ b/flink-streaming-connectors/flink-connector-kafka-0.10/pom.xml
@@ -0,0 +1,179 @@
+
+
+http://maven.apache.org/POM/4.0.0;
+xmlns:xsi="http://www.w3.org/2001/XMLSchema-instance;
+xsi:schemaLocation="http://maven.apache.org/POM/4.0.0 
http://maven.apache.org/maven-v4_0_0.xsd;>
+
+   4.0.0
+
+   
+   org.apache.flink
+   flink-streaming-connectors
+   1.1-SNAPSHOT
+   ..
+   
+
+   flink-connector-kafka-0.10_2.10
+   flink-connector-kafka-0.10
+
+   jar
+
+   
+   
+   0.10.0.0
+   
+
+   
+
+   
+
+   
+   org.apache.flink
+   flink-streaming-java_2.10
+   ${project.version}
+   provided
+   
+
+   
+   org.apache.flink
+   flink-connector-kafka-base_2.10
+   ${project.version}
+   
+   
+   org.apache.kafka
+   
kafka_${scala.binary.version}
+   
+   
+   
+
+   
+   org.apache.flink
+   flink-table_2.10
+   ${project.version}
+   provided
+   
+   true
+   
+
+   
+   org.apache.kafka
+   kafka-clients
+   ${kafka.version}
+   
+
+   
+
+   
+   org.apache.flink
+   flink-connector-kafka-base_2.10
+   ${project.version}
+   
+   
+   
+   org.apache.kafka
+   
kafka_${scala.binary.version}
+   
+   
+   test-jar
+   test
+   
+
+   
+   
+   org.apache.kafka
+   kafka_${scala.binary.version}
+   ${kafka.version}
+   test
+   
+   
+   
+   org.apache.flink
+   flink-tests_2.10
+   ${project.version}
+   test-jar
+   test
+   
+
+   
+   org.apache.flink
+   flink-test-utils_2.10
+   ${project.version}
+   test
+   
+
+   
+