You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@flink.apache.org by se...@apache.org on 2017/02/20 11:06:57 UTC

[1/9] flink git commit: [FLINK-5739] [client] Fix NullPointerException in CliFrontend

Repository: flink
Updated Branches:
  refs/heads/release-1.2 fee020c42 -> 048a21c09


[FLINK-5739] [client] Fix NullPointerException in CliFrontend

This closes #3292


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

Branch: refs/heads/release-1.2
Commit: a082335f993fa6d6dea8820027cfeb6177f94d4b
Parents: 6955030
Author: Zhuoluo Yang <zh...@alibaba-inc.com>
Authored: Tue Feb 14 09:54:50 2017 +0800
Committer: Stephan Ewen <se...@apache.org>
Committed: Mon Feb 20 02:07:54 2017 +0100

----------------------------------------------------------------------
 .../src/main/java/org/apache/flink/client/CliFrontend.java     | 6 ++++++
 .../flink/optimizer/plantranslate/JobGraphGenerator.java       | 3 ++-
 2 files changed, 8 insertions(+), 1 deletion(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/flink/blob/a082335f/flink-clients/src/main/java/org/apache/flink/client/CliFrontend.java
----------------------------------------------------------------------
diff --git a/flink-clients/src/main/java/org/apache/flink/client/CliFrontend.java b/flink-clients/src/main/java/org/apache/flink/client/CliFrontend.java
index 1ec0674..35ed07f 100644
--- a/flink-clients/src/main/java/org/apache/flink/client/CliFrontend.java
+++ b/flink-clients/src/main/java/org/apache/flink/client/CliFrontend.java
@@ -839,6 +839,12 @@ public class CliFrontend {
 			program.deleteExtractedLibraries();
 		}
 
+		if (null == result) {
+			logAndSysout("No JobSubmissionResult returned, please make sure you called " +
+				"ExecutionEnvironment.execute()");
+			return 1;
+		}
+
 		if (result.isJobExecutionResult()) {
 			logAndSysout("Program execution finished");
 			JobExecutionResult execResult = result.getJobExecutionResult();

http://git-wip-us.apache.org/repos/asf/flink/blob/a082335f/flink-optimizer/src/main/java/org/apache/flink/optimizer/plantranslate/JobGraphGenerator.java
----------------------------------------------------------------------
diff --git a/flink-optimizer/src/main/java/org/apache/flink/optimizer/plantranslate/JobGraphGenerator.java b/flink-optimizer/src/main/java/org/apache/flink/optimizer/plantranslate/JobGraphGenerator.java
index 4ccfae3..6f7b04a 100644
--- a/flink-optimizer/src/main/java/org/apache/flink/optimizer/plantranslate/JobGraphGenerator.java
+++ b/flink-optimizer/src/main/java/org/apache/flink/optimizer/plantranslate/JobGraphGenerator.java
@@ -172,7 +172,8 @@ public class JobGraphGenerator implements Visitor<PlanNode> {
 	
 	public JobGraph compileJobGraph(OptimizedPlan program, JobID jobId) {
 		if (program == null) {
-			throw new NullPointerException();
+			throw new NullPointerException("Program is null, did you called " +
+				"ExecutionEnvironment.execute()");
 		}
 		
 		if (jobId == null) {


[6/9] flink git commit: [hotfix] [tests] Use random actor names in JobManagerHARecoveryTest to avoid name collisions

Posted by se...@apache.org.
[hotfix] [tests] Use random actor names in JobManagerHARecoveryTest to avoid name collisions


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

Branch: refs/heads/release-1.2
Commit: 048a21c097060c78e4ef2552c8c774ff56065099
Parents: 7cdbd3c
Author: Stephan Ewen <se...@apache.org>
Authored: Fri Feb 17 20:01:25 2017 +0100
Committer: Stephan Ewen <se...@apache.org>
Committed: Mon Feb 20 02:07:55 2017 +0100

----------------------------------------------------------------------
 .../flink/runtime/jobmanager/JobManagerHARecoveryTest.java   | 8 +++-----
 1 file changed, 3 insertions(+), 5 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/flink/blob/048a21c0/flink-runtime/src/test/java/org/apache/flink/runtime/jobmanager/JobManagerHARecoveryTest.java
----------------------------------------------------------------------
diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/jobmanager/JobManagerHARecoveryTest.java b/flink-runtime/src/test/java/org/apache/flink/runtime/jobmanager/JobManagerHARecoveryTest.java
index 398505f..b8d1d8b 100644
--- a/flink-runtime/src/test/java/org/apache/flink/runtime/jobmanager/JobManagerHARecoveryTest.java
+++ b/flink-runtime/src/test/java/org/apache/flink/runtime/jobmanager/JobManagerHARecoveryTest.java
@@ -171,9 +171,7 @@ public class JobManagerHARecoveryTest {
 			InstanceManager instanceManager = new InstanceManager();
 			instanceManager.addInstanceListener(scheduler);
 
-			archive = system.actorOf(Props.create(
-					MemoryArchivist.class,
-					10), "archive");
+			archive = system.actorOf(Props.create(MemoryArchivist.class, 10));
 
 			executor = new ForkJoinPool();
 
@@ -194,7 +192,7 @@ public class JobManagerHARecoveryTest {
 				jobRecoveryTimeout,
 				Option.apply(null));
 
-			jobManager = system.actorOf(jobManagerProps, "jobmanager");
+			jobManager = system.actorOf(jobManagerProps);
 			ActorGateway gateway = new AkkaActorGateway(jobManager, leaderSessionID);
 
 			taskManager = TaskManager.startTaskManagerComponentsAndActor(
@@ -368,7 +366,7 @@ public class JobManagerHARecoveryTest {
 				Option.<MetricRegistry>apply(null),
 				recoveredJobs).withDispatcher(CallingThreadDispatcher.Id());
 
-			jobManager = system.actorOf(jobManagerProps, "jobmanager");
+			jobManager = system.actorOf(jobManagerProps);
 
 			Future<Object> started = Patterns.ask(jobManager, new Identify(42), deadline.timeLeft().toMillis());
 


[4/9] flink git commit: [FLINK-5817] [test] (followup) Fix temporary folder and temp file path generation

Posted by se...@apache.org.
[FLINK-5817] [test] (followup) Fix temporary folder and temp file path generation

This makes sure the TemporaryFolder rule is already evaluated by the time
the temp files are generated.

This also injects a random parent directory to ensure that even for fix directory/file
names, the absolute path is randomized.


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

Branch: refs/heads/release-1.2
Commit: fc5a20a81f71b81850292bba2643ef37acfb6c2f
Parents: d1b148c
Author: Stephan Ewen <se...@apache.org>
Authored: Sun Feb 19 16:21:07 2017 +0100
Committer: Stephan Ewen <se...@apache.org>
Committed: Mon Feb 20 02:07:54 2017 +0100

----------------------------------------------------------------------
 .../org/apache/flink/test/util/AbstractTestBase.java   | 13 ++++++-------
 .../java/org/apache/flink/test/util/TestBaseUtils.java |  4 ++++
 2 files changed, 10 insertions(+), 7 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/flink/blob/fc5a20a8/flink-test-utils-parent/flink-test-utils/src/main/java/org/apache/flink/test/util/AbstractTestBase.java
----------------------------------------------------------------------
diff --git a/flink-test-utils-parent/flink-test-utils/src/main/java/org/apache/flink/test/util/AbstractTestBase.java b/flink-test-utils-parent/flink-test-utils/src/main/java/org/apache/flink/test/util/AbstractTestBase.java
index 544d473..be9b0b7 100644
--- a/flink-test-utils-parent/flink-test-utils/src/main/java/org/apache/flink/test/util/AbstractTestBase.java
+++ b/flink-test-utils-parent/flink-test-utils/src/main/java/org/apache/flink/test/util/AbstractTestBase.java
@@ -24,7 +24,7 @@ import com.google.common.io.Files;
 import org.apache.flink.configuration.Configuration;
 import org.apache.flink.runtime.minicluster.LocalFlinkMiniCluster;
 
-import org.junit.Rule;
+import org.junit.ClassRule;
 import org.junit.rules.TemporaryFolder;
 
 import scala.concurrent.duration.FiniteDuration;
@@ -49,8 +49,8 @@ public abstract class AbstractTestBase extends TestBaseUtils {
 
 	protected int numTaskManagers = 1;
 
-	@Rule
-	public final TemporaryFolder temporaryFolder = new TemporaryFolder();
+	@ClassRule
+	public static final TemporaryFolder temporaryFolder = new TemporaryFolder();
 
 
 	/** The mini cluster that runs the test programs */
@@ -106,12 +106,12 @@ public abstract class AbstractTestBase extends TestBaseUtils {
 	// --------------------------------------------------------------------------------------------
 
 	public String getTempDirPath(String dirName) throws IOException {
-		File f = temporaryFolder.newFolder(dirName);
+		File f = createAndRegisterTempFile(dirName);
 		return f.toURI().toString();
 	}
 
 	public String getTempFilePath(String fileName) throws IOException {
-		File f = temporaryFolder.newFile(fileName);
+		File f = createAndRegisterTempFile(fileName);
 		return f.toURI().toString();
 	}
 
@@ -122,7 +122,6 @@ public abstract class AbstractTestBase extends TestBaseUtils {
 	}
 
 	public File createAndRegisterTempFile(String fileName) throws IOException {
-		return temporaryFolder.newFile(fileName);
+		return new File(temporaryFolder.newFolder(), fileName);
 	}
-
 }

http://git-wip-us.apache.org/repos/asf/flink/blob/fc5a20a8/flink-test-utils-parent/flink-test-utils/src/main/java/org/apache/flink/test/util/TestBaseUtils.java
----------------------------------------------------------------------
diff --git a/flink-test-utils-parent/flink-test-utils/src/main/java/org/apache/flink/test/util/TestBaseUtils.java b/flink-test-utils-parent/flink-test-utils/src/main/java/org/apache/flink/test/util/TestBaseUtils.java
index 8431226..cc7c0e2 100644
--- a/flink-test-utils-parent/flink-test-utils/src/main/java/org/apache/flink/test/util/TestBaseUtils.java
+++ b/flink-test-utils-parent/flink-test-utils/src/main/java/org/apache/flink/test/util/TestBaseUtils.java
@@ -74,6 +74,8 @@ import java.util.concurrent.TimeUnit;
 import java.util.regex.Matcher;
 import java.util.regex.Pattern;
 
+import static org.apache.flink.util.Preconditions.checkArgument;
+
 import static org.junit.Assert.assertEquals;
 import static org.junit.Assert.assertTrue;
 import static org.junit.Assert.fail;
@@ -283,6 +285,8 @@ public class TestBaseUtils extends TestLogger {
 			String resultPath,
 			String[] excludePrefixes,
 			boolean inOrderOfFiles) throws IOException {
+		
+		checkArgument(resultPath != null, "resultPath cannot be be null");
 
 		final BufferedReader[] readers = getResultReader(resultPath, excludePrefixes, inOrderOfFiles);
 		try {


[2/9] flink git commit: [FLINK-5669] [contrib] Change DataStreamUtils to use the loopback address (127.0.0.1) with local environments.

Posted by se...@apache.org.
[FLINK-5669] [contrib] Change DataStreamUtils to use the loopback address (127.0.0.1) with local environments.

Using loopback rather than the "local address" allows tests to run in
situations where the local machine's hostname may not be resolvable in DNS
(because DNS is unreacable or the hostname is not found) or the hostname does
resolve, but not to an IP address that is reachable.

This closes #3223


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

Branch: refs/heads/release-1.2
Commit: 6955030d5faaf3ffb6156171eed38f10c254295d
Parents: efa5e9d
Author: Rick Cox <ri...@amazon.com>
Authored: Thu Jan 26 14:55:23 2017 -0800
Committer: Stephan Ewen <se...@apache.org>
Committed: Mon Feb 20 02:07:54 2017 +0100

----------------------------------------------------------------------
 .../java/org/apache/flink/contrib/streaming/DataStreamUtils.java  | 3 +++
 1 file changed, 3 insertions(+)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/flink/blob/6955030d/flink-contrib/flink-streaming-contrib/src/main/java/org/apache/flink/contrib/streaming/DataStreamUtils.java
----------------------------------------------------------------------
diff --git a/flink-contrib/flink-streaming-contrib/src/main/java/org/apache/flink/contrib/streaming/DataStreamUtils.java b/flink-contrib/flink-streaming-contrib/src/main/java/org/apache/flink/contrib/streaming/DataStreamUtils.java
index d4ef9ee..2987597 100644
--- a/flink-contrib/flink-streaming-contrib/src/main/java/org/apache/flink/contrib/streaming/DataStreamUtils.java
+++ b/flink-contrib/flink-streaming-contrib/src/main/java/org/apache/flink/contrib/streaming/DataStreamUtils.java
@@ -21,6 +21,7 @@ import org.apache.flink.api.common.typeutils.TypeSerializer;
 import org.apache.flink.runtime.net.ConnectionUtils;
 import org.apache.flink.streaming.api.datastream.DataStream;
 import org.apache.flink.streaming.api.datastream.DataStreamSink;
+import org.apache.flink.streaming.api.environment.LocalStreamEnvironment;
 import org.apache.flink.streaming.api.environment.RemoteStreamEnvironment;
 import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment;
 
@@ -57,6 +58,8 @@ public final class DataStreamUtils {
 				throw new IOException("Could not determine an suitable network address to " +
 						"receive back data from the streaming program.", e);
 			}
+		} else if (env instanceof LocalStreamEnvironment) {
+			clientAddress = InetAddress.getLoopbackAddress();
 		} else {
 			try {
 				clientAddress = InetAddress.getLocalHost();


[5/9] flink git commit: [FLINK-5817] [tests] Use TemporaryFold to create temp files and folds for test

Posted by se...@apache.org.
[FLINK-5817] [tests] Use TemporaryFold to create temp files and folds for test


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

Branch: refs/heads/release-1.2
Commit: d1b148cf30ebd10815912bca2f2b94b2b56a2388
Parents: fee020c
Author: wenlong.lwl <we...@alibaba-inc.com>
Authored: Fri Feb 17 17:14:54 2017 +0800
Committer: Stephan Ewen <se...@apache.org>
Committed: Mon Feb 20 02:07:54 2017 +0100

----------------------------------------------------------------------
 .../io/disk/iomanager/IOManagerTest.java        | 11 ++--
 .../flink/test/util/AbstractTestBase.java       | 53 ++++++--------------
 2 files changed, 22 insertions(+), 42 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/flink/blob/d1b148cf/flink-runtime/src/test/java/org/apache/flink/runtime/io/disk/iomanager/IOManagerTest.java
----------------------------------------------------------------------
diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/io/disk/iomanager/IOManagerTest.java b/flink-runtime/src/test/java/org/apache/flink/runtime/io/disk/iomanager/IOManagerTest.java
index 39cb8ee..156098e 100644
--- a/flink-runtime/src/test/java/org/apache/flink/runtime/io/disk/iomanager/IOManagerTest.java
+++ b/flink-runtime/src/test/java/org/apache/flink/runtime/io/disk/iomanager/IOManagerTest.java
@@ -21,7 +21,9 @@ package org.apache.flink.runtime.io.disk.iomanager;
 import org.apache.flink.core.memory.MemorySegment;
 import org.apache.flink.runtime.io.disk.iomanager.FileIOChannel.ID;
 import org.apache.flink.runtime.io.network.buffer.Buffer;
+import org.junit.Rule;
 import org.junit.Test;
+import org.junit.rules.TemporaryFolder;
 
 import java.io.File;
 import java.io.IOException;
@@ -34,12 +36,15 @@ import static org.junit.Assert.assertTrue;
 
 public class IOManagerTest {
 
+	@Rule
+	public final TemporaryFolder  temporaryFolder = new TemporaryFolder();
+
 	@Test
-	public void channelEnumerator() {
+	public void channelEnumerator() throws IOException {
 		IOManager ioMan = null;
 
 		try {
-			File tempPath = new File(System.getProperty("java.io.tmpdir"));
+			File tempPath = temporaryFolder.newFolder();
 
 			String[] tempDirs = new String[]{
 					new File(tempPath, "a").getAbsolutePath(),
@@ -126,4 +131,4 @@ public class IOManagerTest {
 			throw new UnsupportedOperationException();
 		}
 	}
-}
\ No newline at end of file
+}

http://git-wip-us.apache.org/repos/asf/flink/blob/d1b148cf/flink-test-utils-parent/flink-test-utils/src/main/java/org/apache/flink/test/util/AbstractTestBase.java
----------------------------------------------------------------------
diff --git a/flink-test-utils-parent/flink-test-utils/src/main/java/org/apache/flink/test/util/AbstractTestBase.java b/flink-test-utils-parent/flink-test-utils/src/main/java/org/apache/flink/test/util/AbstractTestBase.java
index 316fd21..544d473 100644
--- a/flink-test-utils-parent/flink-test-utils/src/main/java/org/apache/flink/test/util/AbstractTestBase.java
+++ b/flink-test-utils-parent/flink-test-utils/src/main/java/org/apache/flink/test/util/AbstractTestBase.java
@@ -20,14 +20,17 @@ package org.apache.flink.test.util;
 
 import com.google.common.base.Charsets;
 import com.google.common.io.Files;
+
 import org.apache.flink.configuration.Configuration;
 import org.apache.flink.runtime.minicluster.LocalFlinkMiniCluster;
+
+import org.junit.Rule;
+import org.junit.rules.TemporaryFolder;
+
 import scala.concurrent.duration.FiniteDuration;
 
 import java.io.File;
 import java.io.IOException;
-import java.util.ArrayList;
-import java.util.List;
 import java.util.Objects;
 
 import org.apache.flink.runtime.akka.AkkaUtils;
@@ -39,22 +42,23 @@ public abstract class AbstractTestBase extends TestBaseUtils {
 	
 	/** Configuration to start the testing cluster with */
 	protected final Configuration config;
-	
-	private final List<File> tempFiles;
-	
+
 	private final FiniteDuration timeout;
 
 	protected int taskManagerNumSlots = 1;
 
 	protected int numTaskManagers = 1;
-	
+
+	@Rule
+	public final TemporaryFolder temporaryFolder = new TemporaryFolder();
+
+
 	/** The mini cluster that runs the test programs */
 	protected LocalFlinkMiniCluster executor;
 	
 
 	public AbstractTestBase(Configuration config) {
 		this.config = Objects.requireNonNull(config);
-		this.tempFiles = new ArrayList<File>();
 
 		timeout = AkkaUtils.getTimeout(config);
 	}
@@ -74,7 +78,6 @@ public abstract class AbstractTestBase extends TestBaseUtils {
 
 	public void stopCluster() throws Exception {
 		stopCluster(executor, timeout);
-		deleteAllTempFiles();
 	}
 
 	//------------------
@@ -103,12 +106,12 @@ public abstract class AbstractTestBase extends TestBaseUtils {
 	// --------------------------------------------------------------------------------------------
 
 	public String getTempDirPath(String dirName) throws IOException {
-		File f = createAndRegisterTempFile(dirName);
+		File f = temporaryFolder.newFolder(dirName);
 		return f.toURI().toString();
 	}
 
 	public String getTempFilePath(String fileName) throws IOException {
-		File f = createAndRegisterTempFile(fileName);
+		File f = temporaryFolder.newFile(fileName);
 		return f.toURI().toString();
 	}
 
@@ -119,35 +122,7 @@ public abstract class AbstractTestBase extends TestBaseUtils {
 	}
 
 	public File createAndRegisterTempFile(String fileName) throws IOException {
-		File baseDir = new File(System.getProperty("java.io.tmpdir"));
-		File f = new File(baseDir, this.getClass().getName() + "-" + fileName);
-
-		if (f.exists()) {
-			deleteRecursively(f);
-		}
-
-		File parentToDelete = f;
-		while (true) {
-			File parent = parentToDelete.getParentFile();
-			if (parent == null) {
-				throw new IOException("Missed temp dir while traversing parents of a temp file.");
-			}
-			if (parent.equals(baseDir)) {
-				break;
-			}
-			parentToDelete = parent;
-		}
-
-		Files.createParentDirs(f);
-		this.tempFiles.add(parentToDelete);
-		return f;
+		return temporaryFolder.newFile(fileName);
 	}
 
-	private void deleteAllTempFiles() throws IOException {
-		for (File f : this.tempFiles) {
-			if (f.exists()) {
-				deleteRecursively(f);
-			}
-		}
-	}
 }


[8/9] flink git commit: [hotfix] [core] Add missing @Internal annotations to classes in flink-core.

Posted by se...@apache.org.
[hotfix] [core] Add missing @Internal annotations to classes in flink-core.


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

Branch: refs/heads/release-1.2
Commit: 7cdbd3c6fcd932d54caa7d1cb74a8027812de387
Parents: 7190944
Author: Fabian Hueske <fh...@apache.org>
Authored: Wed Feb 15 15:25:15 2017 +0100
Committer: Stephan Ewen <se...@apache.org>
Committed: Mon Feb 20 02:07:55 2017 +0100

----------------------------------------------------------------------
 .../flink/api/common/ArchivedExecutionConfig.java   |  3 +++
 .../typeutils/TypeSerializerSerializationProxy.java |  4 +++-
 .../api/java/typeutils/runtime/AvroSerializer.java  |  2 ++
 .../typeutils/runtime/CopyableValueComparator.java  |  2 ++
 .../typeutils/runtime/CopyableValueSerializer.java  |  2 ++
 .../java/typeutils/runtime/DataInputDecoder.java    |  3 ++-
 .../java/typeutils/runtime/DataInputViewStream.java |  2 ++
 .../java/typeutils/runtime/DataOutputEncoder.java   |  3 ++-
 .../typeutils/runtime/DataOutputViewStream.java     |  2 ++
 .../java/typeutils/runtime/EitherSerializer.java    | 16 +++++++++-------
 .../api/java/typeutils/runtime/FieldSerializer.java |  3 +++
 .../typeutils/runtime/GenericTypeComparator.java    |  2 ++
 .../api/java/typeutils/runtime/NoFetchingInput.java |  2 ++
 .../java/typeutils/runtime/NullAwareComparator.java |  2 ++
 .../api/java/typeutils/runtime/NullMaskUtils.java   |  2 ++
 .../api/java/typeutils/runtime/PojoComparator.java  |  3 ++-
 .../api/java/typeutils/runtime/PojoSerializer.java  |  2 ++
 .../api/java/typeutils/runtime/RowComparator.java   |  2 ++
 .../api/java/typeutils/runtime/RowSerializer.java   |  2 ++
 .../typeutils/runtime/RuntimeComparatorFactory.java |  2 ++
 .../runtime/RuntimePairComparatorFactory.java       |  2 ++
 .../typeutils/runtime/RuntimeSerializerFactory.java |  2 ++
 .../java/typeutils/runtime/Tuple0Serializer.java    |  2 ++
 .../api/java/typeutils/runtime/TupleComparator.java |  3 ++-
 .../java/typeutils/runtime/TupleComparatorBase.java |  3 ++-
 .../api/java/typeutils/runtime/TupleSerializer.java |  3 ++-
 .../java/typeutils/runtime/TupleSerializerBase.java |  2 ++
 .../api/java/typeutils/runtime/ValueComparator.java |  2 ++
 .../api/java/typeutils/runtime/ValueSerializer.java |  2 ++
 .../core/fs/AbstractMultiFSDataInputStream.java     |  2 ++
 .../flink/core/fs/ClosingFSDataInputStream.java     |  4 +++-
 .../flink/core/fs/ClosingFSDataOutputStream.java    |  4 +++-
 .../flink/core/fs/FSDataInputStreamWrapper.java     |  4 +++-
 .../flink/core/fs/FSDataOutputStreamWrapper.java    |  4 +++-
 .../flink/core/fs/SafetyNetWrapperFileSystem.java   |  4 +++-
 .../flink/core/fs/WrappingProxyCloseable.java       |  2 ++
 .../core/memory/ByteArrayOutputStreamWithPos.java   |  2 ++
 .../flink/util/AbstractCloseableRegistry.java       |  3 +++
 .../java/org/apache/flink/util/CollectionUtil.java  |  3 +++
 .../main/java/org/apache/flink/util/FutureUtil.java |  5 ++++-
 .../main/java/org/apache/flink/util/Migration.java  |  3 +++
 .../java/org/apache/flink/util/WrappingProxy.java   |  3 +++
 .../org/apache/flink/util/WrappingProxyUtil.java    |  5 ++++-
 43 files changed, 109 insertions(+), 21 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/api/common/ArchivedExecutionConfig.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/api/common/ArchivedExecutionConfig.java b/flink-core/src/main/java/org/apache/flink/api/common/ArchivedExecutionConfig.java
index faf920d..f267e91 100644
--- a/flink-core/src/main/java/org/apache/flink/api/common/ArchivedExecutionConfig.java
+++ b/flink-core/src/main/java/org/apache/flink/api/common/ArchivedExecutionConfig.java
@@ -17,6 +17,8 @@
  */
 package org.apache.flink.api.common;
 
+import org.apache.flink.annotation.Internal;
+
 import java.io.Serializable;
 import java.util.Collections;
 import java.util.Map;
@@ -26,6 +28,7 @@ import java.util.Map;
  * It can be used to display job information on the web interface
  * without having to keep the classloader around after job completion.
  */
+@Internal
 public class ArchivedExecutionConfig implements Serializable {
 
 	private final String executionMode;

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/api/common/typeutils/TypeSerializerSerializationProxy.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/api/common/typeutils/TypeSerializerSerializationProxy.java b/flink-core/src/main/java/org/apache/flink/api/common/typeutils/TypeSerializerSerializationProxy.java
index 06ad8bf..cebd348 100644
--- a/flink-core/src/main/java/org/apache/flink/api/common/typeutils/TypeSerializerSerializationProxy.java
+++ b/flink-core/src/main/java/org/apache/flink/api/common/typeutils/TypeSerializerSerializationProxy.java
@@ -18,6 +18,7 @@
 
 package org.apache.flink.api.common.typeutils;
 
+import org.apache.flink.annotation.Internal;
 import org.apache.flink.core.io.VersionedIOReadableWritable;
 import org.apache.flink.core.memory.ByteArrayOutputStreamWithPos;
 import org.apache.flink.core.memory.DataInputView;
@@ -30,6 +31,7 @@ import org.slf4j.LoggerFactory;
 import java.io.IOException;
 import java.util.Arrays;
 
+@Internal
 public class TypeSerializerSerializationProxy<T> extends VersionedIOReadableWritable {
 
 	public static final int VERSION = 1;
@@ -228,4 +230,4 @@ public class TypeSerializerSerializationProxy<T> extends VersionedIOReadableWrit
 			return Arrays.hashCode(getActualBytes());
 		}
 	}
-}
\ No newline at end of file
+}

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/AvroSerializer.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/AvroSerializer.java b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/AvroSerializer.java
index 4c2a7f9..8bcd157 100644
--- a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/AvroSerializer.java
+++ b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/AvroSerializer.java
@@ -25,6 +25,7 @@ import org.apache.avro.reflect.ReflectDatumReader;
 import org.apache.avro.reflect.ReflectDatumWriter;
 import org.apache.avro.util.Utf8;
 
+import org.apache.flink.annotation.Internal;
 import org.apache.flink.api.common.typeutils.TypeSerializer;
 import org.apache.flink.api.java.typeutils.runtime.kryo.Serializers;
 import org.apache.flink.core.memory.DataInputView;
@@ -42,6 +43,7 @@ import static org.apache.flink.util.Preconditions.checkNotNull;
  *
  * @param <T> The type serialized.
  */
+@Internal
 public final class AvroSerializer<T> extends TypeSerializer<T> {
 
 	private static final long serialVersionUID = 1L;

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/CopyableValueComparator.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/CopyableValueComparator.java b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/CopyableValueComparator.java
index 9b3b191..0093436 100644
--- a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/CopyableValueComparator.java
+++ b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/CopyableValueComparator.java
@@ -20,6 +20,7 @@ package org.apache.flink.api.java.typeutils.runtime;
 
 import java.io.IOException;
 
+import org.apache.flink.annotation.Internal;
 import org.apache.flink.api.common.typeutils.TypeComparator;
 import org.apache.flink.core.memory.DataInputView;
 import org.apache.flink.core.memory.DataOutputView;
@@ -31,6 +32,7 @@ import org.apache.flink.util.InstantiationUtil;
 /**
  * Comparator for all Value types that extend Key
  */
+@Internal
 public class CopyableValueComparator<T extends CopyableValue<T> & Comparable<T>> extends TypeComparator<T> {
 	
 	private static final long serialVersionUID = 1L;

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/CopyableValueSerializer.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/CopyableValueSerializer.java b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/CopyableValueSerializer.java
index f30a767..1f3fcbc 100644
--- a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/CopyableValueSerializer.java
+++ b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/CopyableValueSerializer.java
@@ -20,6 +20,7 @@ package org.apache.flink.api.java.typeutils.runtime;
 
 import java.io.IOException;
 
+import org.apache.flink.annotation.Internal;
 import org.apache.flink.api.common.typeutils.TypeSerializer;
 import org.apache.flink.core.memory.DataInputView;
 import org.apache.flink.core.memory.DataOutputView;
@@ -28,6 +29,7 @@ import org.apache.flink.util.InstantiationUtil;
 
 import static org.apache.flink.util.Preconditions.checkNotNull;
 
+@Internal
 public class CopyableValueSerializer<T extends CopyableValue<T>> extends TypeSerializer<T> {
 
 	private static final long serialVersionUID = 1L;

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/DataInputDecoder.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/DataInputDecoder.java b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/DataInputDecoder.java
index e48f9fa..c0454c6 100644
--- a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/DataInputDecoder.java
+++ b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/DataInputDecoder.java
@@ -24,8 +24,9 @@ import java.nio.ByteBuffer;
 
 import org.apache.avro.io.Decoder;
 import org.apache.avro.util.Utf8;
+import org.apache.flink.annotation.Internal;
 
-
+@Internal
 public class DataInputDecoder extends Decoder implements java.io.Serializable {
 	
 	private static final long serialVersionUID = 1L;

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/DataInputViewStream.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/DataInputViewStream.java b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/DataInputViewStream.java
index be17d64..aab4689 100644
--- a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/DataInputViewStream.java
+++ b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/DataInputViewStream.java
@@ -18,6 +18,7 @@
 
 package org.apache.flink.api.java.typeutils.runtime;
 
+import org.apache.flink.annotation.Internal;
 import org.apache.flink.core.memory.DataInputView;
 
 import java.io.EOFException;
@@ -27,6 +28,7 @@ import java.io.InputStream;
 /**
  * An input stream that draws its data from a {@link DataInputView}.
  */
+@Internal
 public class DataInputViewStream extends InputStream {
 	
 	protected DataInputView inputView;

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/DataOutputEncoder.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/DataOutputEncoder.java b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/DataOutputEncoder.java
index 5c89962..c41b648 100644
--- a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/DataOutputEncoder.java
+++ b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/DataOutputEncoder.java
@@ -24,8 +24,9 @@ import java.nio.ByteBuffer;
 
 import org.apache.avro.io.Encoder;
 import org.apache.avro.util.Utf8;
+import org.apache.flink.annotation.Internal;
 
-
+@Internal
 public final class DataOutputEncoder extends Encoder implements java.io.Serializable {
 	
 	private static final long serialVersionUID = 1L;

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/DataOutputViewStream.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/DataOutputViewStream.java b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/DataOutputViewStream.java
index 66f2af6..9c06043 100644
--- a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/DataOutputViewStream.java
+++ b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/DataOutputViewStream.java
@@ -18,10 +18,12 @@
 
 package org.apache.flink.api.java.typeutils.runtime;
 
+import org.apache.flink.annotation.Internal;
 import org.apache.flink.core.memory.DataOutputView;
 import java.io.IOException;
 import java.io.OutputStream;
 
+@Internal
 public class DataOutputViewStream extends OutputStream {
 	protected DataOutputView outputView;
 

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/EitherSerializer.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/EitherSerializer.java b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/EitherSerializer.java
index 7c9676b..4066e9a 100644
--- a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/EitherSerializer.java
+++ b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/EitherSerializer.java
@@ -18,22 +18,24 @@
 
 package org.apache.flink.api.java.typeutils.runtime;
 
-import static org.apache.flink.types.Either.Left;
-import static org.apache.flink.types.Either.Right;
-
-import java.io.IOException;
-
+import org.apache.flink.annotation.Internal;
 import org.apache.flink.api.common.typeutils.TypeSerializer;
-import org.apache.flink.types.Either;
 import org.apache.flink.core.memory.DataInputView;
 import org.apache.flink.core.memory.DataOutputView;
+import org.apache.flink.types.Either;
+
+import java.io.IOException;
+
+import static org.apache.flink.types.Either.Left;
+import static org.apache.flink.types.Either.Right;
 
 /**
- * A {@link TypeSerializer} for the {@ link Either} type of the Java class.
+ * A {@link TypeSerializer} for the {@link Either} type of the Java class.
  *
  * @param <L> the Left value type
  * @param <R> the Right value type
  */
+@Internal
 public class EitherSerializer<L, R> extends TypeSerializer<Either<L, R>> {
 
 	private static final long serialVersionUID = 1L;

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/FieldSerializer.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/FieldSerializer.java b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/FieldSerializer.java
index 057eee9..5519889 100644
--- a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/FieldSerializer.java
+++ b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/FieldSerializer.java
@@ -18,6 +18,8 @@
 
 package org.apache.flink.api.java.typeutils.runtime;
 
+import org.apache.flink.annotation.Internal;
+
 import java.io.IOException;
 import java.io.ObjectInputStream;
 import java.io.ObjectOutputStream;
@@ -28,6 +30,7 @@ import java.lang.reflect.Field;
  * readObject/writeObject need to be implemented in classes where there is a field of type java.lang.reflect.Field.
  * The two static methods in this class are to be called from these readObject/writeObject methods.
  */
+@Internal
 public class FieldSerializer {
 
 	public static void serializeField(Field field, ObjectOutputStream out) throws IOException {

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/GenericTypeComparator.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/GenericTypeComparator.java b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/GenericTypeComparator.java
index 28fea6a..e708079 100644
--- a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/GenericTypeComparator.java
+++ b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/GenericTypeComparator.java
@@ -21,6 +21,7 @@ package org.apache.flink.api.java.typeutils.runtime;
 
 import java.io.IOException;
 
+import org.apache.flink.annotation.Internal;
 import org.apache.flink.api.common.typeutils.TypeComparator;
 import org.apache.flink.api.common.typeutils.TypeSerializer;
 import org.apache.flink.core.memory.DataInputView;
@@ -32,6 +33,7 @@ import org.apache.flink.util.InstantiationUtil;
 /**
  * TypeComparator for all types that extend Comparable.
  */
+@Internal
 public class GenericTypeComparator<T extends Comparable<T>> extends TypeComparator<T> {
 
 	private static final long serialVersionUID = 1L;

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/NoFetchingInput.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/NoFetchingInput.java b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/NoFetchingInput.java
index 0f4fe94..b247783 100644
--- a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/NoFetchingInput.java
+++ b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/NoFetchingInput.java
@@ -20,11 +20,13 @@ package org.apache.flink.api.java.typeutils.runtime;
 
 import com.esotericsoftware.kryo.KryoException;
 import com.esotericsoftware.kryo.io.Input;
+import org.apache.flink.annotation.Internal;
 
 import java.io.EOFException;
 import java.io.IOException;
 import java.io.InputStream;
 
+@Internal
 public class NoFetchingInput extends Input {
 	public NoFetchingInput(InputStream inputStream){
 		super(inputStream, 8);

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/NullAwareComparator.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/NullAwareComparator.java b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/NullAwareComparator.java
index 3587811..47297c7 100644
--- a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/NullAwareComparator.java
+++ b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/NullAwareComparator.java
@@ -17,6 +17,7 @@
  */
 package org.apache.flink.api.java.typeutils.runtime;
 
+import org.apache.flink.annotation.Internal;
 import org.apache.flink.api.common.typeutils.CompositeTypeComparator;
 import org.apache.flink.api.common.typeutils.TypeComparator;
 import org.apache.flink.core.memory.DataInputView;
@@ -33,6 +34,7 @@ import java.util.List;
  * NOTE: This class assumes to be used within a composite type comparator (such
  * as {@link RowComparator}) that handles serialized comparison.
  */
+@Internal
 public class NullAwareComparator<T> extends TypeComparator<T> {
 	private static final long serialVersionUID = 1L;
 

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/NullMaskUtils.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/NullMaskUtils.java b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/NullMaskUtils.java
index 010af7f..cfe562f 100644
--- a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/NullMaskUtils.java
+++ b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/NullMaskUtils.java
@@ -17,12 +17,14 @@
  */
 package org.apache.flink.api.java.typeutils.runtime;
 
+import org.apache.flink.annotation.Internal;
 import org.apache.flink.core.memory.DataInputView;
 import org.apache.flink.core.memory.DataOutputView;
 import org.apache.flink.types.Row;
 
 import java.io.IOException;
 
+@Internal
 public class NullMaskUtils {
 
 	public static void writeNullMask(int len, Row value, DataOutputView target) throws IOException {

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/PojoComparator.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/PojoComparator.java b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/PojoComparator.java
index fc4a305..945abc8 100644
--- a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/PojoComparator.java
+++ b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/PojoComparator.java
@@ -24,6 +24,7 @@ import java.io.ObjectOutputStream;
 import java.lang.reflect.Field;
 import java.util.List;
 
+import org.apache.flink.annotation.Internal;
 import org.apache.flink.api.common.typeutils.CompositeTypeComparator;
 import org.apache.flink.api.common.typeutils.TypeComparator;
 import org.apache.flink.api.common.typeutils.TypeSerializer;
@@ -33,7 +34,7 @@ import org.apache.flink.core.memory.MemorySegment;
 import org.apache.flink.types.NullKeyFieldException;
 import org.apache.flink.util.InstantiationUtil;
 
-
+@Internal
 public final class PojoComparator<T> extends CompositeTypeComparator<T> implements java.io.Serializable {
 	
 	private static final long serialVersionUID = 1L;

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/PojoSerializer.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/PojoSerializer.java b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/PojoSerializer.java
index 57928b8..1a9c8f9 100644
--- a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/PojoSerializer.java
+++ b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/PojoSerializer.java
@@ -32,6 +32,7 @@ import java.util.List;
 import java.util.Map;
 import java.util.Objects;
 
+import org.apache.flink.annotation.Internal;
 import org.apache.flink.api.common.ExecutionConfig;
 import org.apache.flink.api.common.typeinfo.TypeInformation;
 import org.apache.flink.api.common.typeutils.TypeSerializer;
@@ -41,6 +42,7 @@ import org.apache.flink.core.memory.DataOutputView;
 
 import static org.apache.flink.util.Preconditions.checkNotNull;
 
+@Internal
 public final class PojoSerializer<T> extends TypeSerializer<T> {
 
 	// Flags for the header

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/RowComparator.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/RowComparator.java b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/RowComparator.java
index d6c5195..135623b 100644
--- a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/RowComparator.java
+++ b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/RowComparator.java
@@ -17,6 +17,7 @@
  */
 package org.apache.flink.api.java.typeutils.runtime;
 
+import org.apache.flink.annotation.Internal;
 import org.apache.flink.api.common.typeutils.CompositeTypeComparator;
 import org.apache.flink.api.common.typeutils.TypeComparator;
 import org.apache.flink.api.common.typeutils.TypeSerializer;
@@ -37,6 +38,7 @@ import static org.apache.flink.util.Preconditions.checkArgument;
 /**
  * Comparator for {@link Row}
  */
+@Internal
 public class RowComparator extends CompositeTypeComparator<Row> {
 
 	private static final long serialVersionUID = 1L;

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/RowSerializer.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/RowSerializer.java b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/RowSerializer.java
index 5457c05..827c2fa 100644
--- a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/RowSerializer.java
+++ b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/RowSerializer.java
@@ -17,6 +17,7 @@
  */
 package org.apache.flink.api.java.typeutils.runtime;
 
+import org.apache.flink.annotation.Internal;
 import org.apache.flink.api.common.typeutils.TypeSerializer;
 import org.apache.flink.core.memory.DataInputView;
 import org.apache.flink.core.memory.DataOutputView;
@@ -33,6 +34,7 @@ import static org.apache.flink.util.Preconditions.checkNotNull;
 /**
  * Serializer for {@link Row}.
  */
+@Internal
 public class RowSerializer extends TypeSerializer<Row> {
 
 	private static final long serialVersionUID = 1L;

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/RuntimeComparatorFactory.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/RuntimeComparatorFactory.java b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/RuntimeComparatorFactory.java
index 4b734a7..13d5575 100644
--- a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/RuntimeComparatorFactory.java
+++ b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/RuntimeComparatorFactory.java
@@ -19,11 +19,13 @@
 
 package org.apache.flink.api.java.typeutils.runtime;
 
+import org.apache.flink.annotation.Internal;
 import org.apache.flink.api.common.typeutils.TypeComparator;
 import org.apache.flink.api.common.typeutils.TypeComparatorFactory;
 import org.apache.flink.configuration.Configuration;
 import org.apache.flink.util.InstantiationUtil;
 
+@Internal
 public final class RuntimeComparatorFactory<T> implements TypeComparatorFactory<T>, java.io.Serializable {
 
 	private static final long serialVersionUID = 1L;

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/RuntimePairComparatorFactory.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/RuntimePairComparatorFactory.java b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/RuntimePairComparatorFactory.java
index 31e28f7..13b3f2c 100644
--- a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/RuntimePairComparatorFactory.java
+++ b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/RuntimePairComparatorFactory.java
@@ -18,11 +18,13 @@
 
 package org.apache.flink.api.java.typeutils.runtime;
 
+import org.apache.flink.annotation.Internal;
 import org.apache.flink.api.common.typeutils.GenericPairComparator;
 import org.apache.flink.api.common.typeutils.TypeComparator;
 import org.apache.flink.api.common.typeutils.TypePairComparator;
 import org.apache.flink.api.common.typeutils.TypePairComparatorFactory;
 
+@Internal
 public final class RuntimePairComparatorFactory<T1, T2>
 		implements TypePairComparatorFactory<T1, T2>, java.io.Serializable {
 

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/RuntimeSerializerFactory.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/RuntimeSerializerFactory.java b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/RuntimeSerializerFactory.java
index 96aff73..f03a7ab 100644
--- a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/RuntimeSerializerFactory.java
+++ b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/RuntimeSerializerFactory.java
@@ -19,11 +19,13 @@
 
 package org.apache.flink.api.java.typeutils.runtime;
 
+import org.apache.flink.annotation.Internal;
 import org.apache.flink.api.common.typeutils.TypeSerializer;
 import org.apache.flink.api.common.typeutils.TypeSerializerFactory;
 import org.apache.flink.configuration.Configuration;
 import org.apache.flink.util.InstantiationUtil;
 
+@Internal
 public final class RuntimeSerializerFactory<T> implements TypeSerializerFactory<T>, java.io.Serializable {
 
 	private static final long serialVersionUID = 1L;

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/Tuple0Serializer.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/Tuple0Serializer.java b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/Tuple0Serializer.java
index a06ff1a..b96126d 100644
--- a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/Tuple0Serializer.java
+++ b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/Tuple0Serializer.java
@@ -14,11 +14,13 @@ package org.apache.flink.api.java.typeutils.runtime;
 
 import java.io.IOException;
 
+import org.apache.flink.annotation.Internal;
 import org.apache.flink.api.common.typeutils.TypeSerializer;
 import org.apache.flink.api.java.tuple.Tuple0;
 import org.apache.flink.core.memory.DataInputView;
 import org.apache.flink.core.memory.DataOutputView;
 
+@Internal
 public class Tuple0Serializer extends TupleSerializer<Tuple0> {
 	
 	private static final long serialVersionUID = 1278813169022975971L;

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/TupleComparator.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/TupleComparator.java b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/TupleComparator.java
index 875ecc2..e5e32be 100644
--- a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/TupleComparator.java
+++ b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/TupleComparator.java
@@ -18,6 +18,7 @@
 
 package org.apache.flink.api.java.typeutils.runtime;
 
+import org.apache.flink.annotation.Internal;
 import org.apache.flink.api.common.typeutils.TypeComparator;
 import org.apache.flink.api.common.typeutils.TypeSerializer;
 import org.apache.flink.api.java.tuple.Tuple;
@@ -26,7 +27,7 @@ import org.apache.flink.types.KeyFieldOutOfBoundsException;
 import org.apache.flink.types.NullFieldException;
 import org.apache.flink.types.NullKeyFieldException;
 
-
+@Internal
 public final class TupleComparator<T extends Tuple> extends TupleComparatorBase<T> {
 
 	private static final long serialVersionUID = 1L;

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/TupleComparatorBase.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/TupleComparatorBase.java b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/TupleComparatorBase.java
index 28169e5..f7a327a 100644
--- a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/TupleComparatorBase.java
+++ b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/TupleComparatorBase.java
@@ -20,6 +20,7 @@ package org.apache.flink.api.java.typeutils.runtime;
 import java.io.IOException;
 import java.util.List;
 
+import org.apache.flink.annotation.Internal;
 import org.apache.flink.api.common.typeutils.CompositeTypeComparator;
 import org.apache.flink.api.common.typeutils.TypeComparator;
 import org.apache.flink.api.common.typeutils.TypeSerializer;
@@ -28,7 +29,7 @@ import org.apache.flink.core.memory.DataOutputView;
 import org.apache.flink.types.KeyFieldOutOfBoundsException;
 import org.apache.flink.types.NullKeyFieldException;
 
-
+@Internal
 public abstract class TupleComparatorBase<T> extends CompositeTypeComparator<T> implements java.io.Serializable {
 
 	private static final long serialVersionUID = 1L;

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/TupleSerializer.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/TupleSerializer.java b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/TupleSerializer.java
index 0897063..df6694c 100644
--- a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/TupleSerializer.java
+++ b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/TupleSerializer.java
@@ -20,13 +20,14 @@ package org.apache.flink.api.java.typeutils.runtime;
 
 import java.io.IOException;
 
+import org.apache.flink.annotation.Internal;
 import org.apache.flink.api.common.typeutils.TypeSerializer;
 import org.apache.flink.api.java.tuple.Tuple;
 import org.apache.flink.core.memory.DataInputView;
 import org.apache.flink.core.memory.DataOutputView;
 import org.apache.flink.types.NullFieldException;
 
-
+@Internal
 public class TupleSerializer<T extends Tuple> extends TupleSerializerBase<T> {
 
 	private static final long serialVersionUID = 1L;

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/TupleSerializerBase.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/TupleSerializerBase.java b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/TupleSerializerBase.java
index 5a93cc5..afc4aa2 100644
--- a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/TupleSerializerBase.java
+++ b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/TupleSerializerBase.java
@@ -18,6 +18,7 @@
 
 package org.apache.flink.api.java.typeutils.runtime;
 
+import org.apache.flink.annotation.Internal;
 import org.apache.flink.api.common.typeutils.TypeSerializer;
 import org.apache.flink.core.memory.DataInputView;
 import org.apache.flink.core.memory.DataOutputView;
@@ -28,6 +29,7 @@ import java.util.Objects;
 
 import static org.apache.flink.util.Preconditions.checkNotNull;
 
+@Internal
 public abstract class TupleSerializerBase<T> extends TypeSerializer<T> {
 
 	private static final long serialVersionUID = 1L;

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/ValueComparator.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/ValueComparator.java b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/ValueComparator.java
index 4b9629a..fb81bcd 100644
--- a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/ValueComparator.java
+++ b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/ValueComparator.java
@@ -20,6 +20,7 @@ package org.apache.flink.api.java.typeutils.runtime;
 
 import java.io.IOException;
 
+import org.apache.flink.annotation.Internal;
 import org.apache.flink.api.common.typeutils.TypeComparator;
 import org.apache.flink.core.memory.DataInputView;
 import org.apache.flink.core.memory.DataOutputView;
@@ -34,6 +35,7 @@ import org.objenesis.strategy.StdInstantiatorStrategy;
 /**
  * Comparator for all Value types that extend Key
  */
+@Internal
 public class ValueComparator<T extends Value & Comparable<T>> extends TypeComparator<T> {
 	
 	private static final long serialVersionUID = 1L;

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/ValueSerializer.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/ValueSerializer.java b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/ValueSerializer.java
index 73dc0fc..56e204c 100644
--- a/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/ValueSerializer.java
+++ b/flink-core/src/main/java/org/apache/flink/api/java/typeutils/runtime/ValueSerializer.java
@@ -20,6 +20,7 @@ package org.apache.flink.api.java.typeutils.runtime;
 
 import java.io.IOException;
 
+import org.apache.flink.annotation.Internal;
 import org.apache.flink.api.common.typeutils.TypeSerializer;
 import org.apache.flink.core.memory.DataInputView;
 import org.apache.flink.core.memory.DataOutputView;
@@ -37,6 +38,7 @@ import static org.apache.flink.util.Preconditions.checkNotNull;
  *
  * @param <T> The type serialized.
  */
+@Internal
 public class ValueSerializer<T extends Value> extends TypeSerializer<T> {
 
 	private static final long serialVersionUID = 1L;

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/core/fs/AbstractMultiFSDataInputStream.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/core/fs/AbstractMultiFSDataInputStream.java b/flink-core/src/main/java/org/apache/flink/core/fs/AbstractMultiFSDataInputStream.java
index 88c0092..a161ceb 100644
--- a/flink-core/src/main/java/org/apache/flink/core/fs/AbstractMultiFSDataInputStream.java
+++ b/flink-core/src/main/java/org/apache/flink/core/fs/AbstractMultiFSDataInputStream.java
@@ -18,6 +18,7 @@
 
 package org.apache.flink.core.fs;
 
+import org.apache.flink.annotation.Internal;
 import org.apache.flink.util.IOUtils;
 import org.apache.flink.util.Preconditions;
 
@@ -28,6 +29,7 @@ import java.io.IOException;
  * Abstract base class for wrappers over multiple {@link FSDataInputStream}, which gives a contiguous view on all inner
  * streams and makes them look like a single stream, in which we can read, seek, etc.
  */
+@Internal
 public abstract class AbstractMultiFSDataInputStream extends FSDataInputStream {
 
 	/** Inner stream for the currently accessed segment of the virtual global stream */

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/core/fs/ClosingFSDataInputStream.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/core/fs/ClosingFSDataInputStream.java b/flink-core/src/main/java/org/apache/flink/core/fs/ClosingFSDataInputStream.java
index 23ac4f2..7c97271 100644
--- a/flink-core/src/main/java/org/apache/flink/core/fs/ClosingFSDataInputStream.java
+++ b/flink-core/src/main/java/org/apache/flink/core/fs/ClosingFSDataInputStream.java
@@ -18,6 +18,7 @@
 
 package org.apache.flink.core.fs;
 
+import org.apache.flink.annotation.Internal;
 import org.apache.flink.util.Preconditions;
 
 import java.io.IOException;
@@ -28,6 +29,7 @@ import java.io.IOException;
  * <p>
  * See {@link SafetyNetCloseableRegistry} for more details on how this is utilized.
  */
+@Internal
 public class ClosingFSDataInputStream
 		extends FSDataInputStreamWrapper
 		implements WrappingProxyCloseable<FSDataInputStream> {
@@ -94,4 +96,4 @@ public class ClosingFSDataInputStream
 		registry.registerClosable(inputStream);
 		return inputStream;
 	}
-}
\ No newline at end of file
+}

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/core/fs/ClosingFSDataOutputStream.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/core/fs/ClosingFSDataOutputStream.java b/flink-core/src/main/java/org/apache/flink/core/fs/ClosingFSDataOutputStream.java
index 120ca67..c517a83 100644
--- a/flink-core/src/main/java/org/apache/flink/core/fs/ClosingFSDataOutputStream.java
+++ b/flink-core/src/main/java/org/apache/flink/core/fs/ClosingFSDataOutputStream.java
@@ -18,6 +18,7 @@
 
 package org.apache.flink.core.fs;
 
+import org.apache.flink.annotation.Internal;
 import org.apache.flink.util.Preconditions;
 
 import java.io.IOException;
@@ -28,6 +29,7 @@ import java.io.IOException;
  * <p>
  * See {@link SafetyNetCloseableRegistry} for more details on how this is utilized.
  */
+@Internal
 public class ClosingFSDataOutputStream
 		extends FSDataOutputStreamWrapper
 		implements WrappingProxyCloseable<FSDataOutputStream> {
@@ -99,4 +101,4 @@ public class ClosingFSDataOutputStream
 		registry.registerClosable(inputStream);
 		return inputStream;
 	}
-}
\ No newline at end of file
+}

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/core/fs/FSDataInputStreamWrapper.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/core/fs/FSDataInputStreamWrapper.java b/flink-core/src/main/java/org/apache/flink/core/fs/FSDataInputStreamWrapper.java
index 507b756..d2eb9f2 100644
--- a/flink-core/src/main/java/org/apache/flink/core/fs/FSDataInputStreamWrapper.java
+++ b/flink-core/src/main/java/org/apache/flink/core/fs/FSDataInputStreamWrapper.java
@@ -18,6 +18,7 @@
 
 package org.apache.flink.core.fs;
 
+import org.apache.flink.annotation.Internal;
 import org.apache.flink.util.Preconditions;
 import org.apache.flink.util.WrappingProxy;
 
@@ -26,6 +27,7 @@ import java.io.IOException;
 /**
  * Simple forwarding wrapper around {@link FSDataInputStream}
  */
+@Internal
 public class FSDataInputStreamWrapper extends FSDataInputStream implements WrappingProxy<FSDataInputStream> {
 
 	protected final FSDataInputStream inputStream;
@@ -93,4 +95,4 @@ public class FSDataInputStreamWrapper extends FSDataInputStream implements Wrapp
 	public FSDataInputStream getWrappedDelegate() {
 		return inputStream;
 	}
-}
\ No newline at end of file
+}

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/core/fs/FSDataOutputStreamWrapper.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/core/fs/FSDataOutputStreamWrapper.java b/flink-core/src/main/java/org/apache/flink/core/fs/FSDataOutputStreamWrapper.java
index 36ebe10..f015012 100644
--- a/flink-core/src/main/java/org/apache/flink/core/fs/FSDataOutputStreamWrapper.java
+++ b/flink-core/src/main/java/org/apache/flink/core/fs/FSDataOutputStreamWrapper.java
@@ -18,6 +18,7 @@
 
 package org.apache.flink.core.fs;
 
+import org.apache.flink.annotation.Internal;
 import org.apache.flink.util.Preconditions;
 import org.apache.flink.util.WrappingProxy;
 
@@ -26,6 +27,7 @@ import java.io.IOException;
 /**
  * Simple forwarding wrapper around {@link FSDataInputStream}
  */
+@Internal
 public class FSDataOutputStreamWrapper extends FSDataOutputStream implements WrappingProxy<FSDataOutputStream> {
 
 	protected final FSDataOutputStream outputStream;
@@ -73,4 +75,4 @@ public class FSDataOutputStreamWrapper extends FSDataOutputStream implements Wra
 	public FSDataOutputStream getWrappedDelegate() {
 		return outputStream;
 	}
-}
\ No newline at end of file
+}

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/core/fs/SafetyNetWrapperFileSystem.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/core/fs/SafetyNetWrapperFileSystem.java b/flink-core/src/main/java/org/apache/flink/core/fs/SafetyNetWrapperFileSystem.java
index bf30b4f..63e6253 100644
--- a/flink-core/src/main/java/org/apache/flink/core/fs/SafetyNetWrapperFileSystem.java
+++ b/flink-core/src/main/java/org/apache/flink/core/fs/SafetyNetWrapperFileSystem.java
@@ -18,6 +18,7 @@
 
 package org.apache.flink.core.fs;
 
+import org.apache.flink.annotation.Internal;
 import org.apache.flink.util.Preconditions;
 import org.apache.flink.util.WrappingProxy;
 
@@ -32,6 +33,7 @@ import java.net.URI;
  * Streams obtained by this are therefore managed by the {@link SafetyNetCloseableRegistry} to prevent resource leaks
  * from unclosed streams.
  */
+@Internal
 public class SafetyNetWrapperFileSystem extends FileSystem implements WrappingProxy<FileSystem> {
 
 	private final SafetyNetCloseableRegistry registry;
@@ -147,4 +149,4 @@ public class SafetyNetWrapperFileSystem extends FileSystem implements WrappingPr
 	public FileSystem getWrappedDelegate() {
 		return unsafeFileSystem;
 	}
-}
\ No newline at end of file
+}

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/core/fs/WrappingProxyCloseable.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/core/fs/WrappingProxyCloseable.java b/flink-core/src/main/java/org/apache/flink/core/fs/WrappingProxyCloseable.java
index b74fc78..9f100ef 100644
--- a/flink-core/src/main/java/org/apache/flink/core/fs/WrappingProxyCloseable.java
+++ b/flink-core/src/main/java/org/apache/flink/core/fs/WrappingProxyCloseable.java
@@ -18,6 +18,7 @@
 
 package org.apache.flink.core.fs;
 
+import org.apache.flink.annotation.Internal;
 import org.apache.flink.util.WrappingProxy;
 
 import java.io.Closeable;
@@ -25,6 +26,7 @@ import java.io.Closeable;
 /**
  * {@link WrappingProxy} for {@link Closeable} that is also closeable.
  */
+@Internal
 public interface WrappingProxyCloseable<T extends Closeable> extends Closeable, WrappingProxy<T> {
 
 }

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/core/memory/ByteArrayOutputStreamWithPos.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/core/memory/ByteArrayOutputStreamWithPos.java b/flink-core/src/main/java/org/apache/flink/core/memory/ByteArrayOutputStreamWithPos.java
index ebaf1b9..ddfd30a 100644
--- a/flink-core/src/main/java/org/apache/flink/core/memory/ByteArrayOutputStreamWithPos.java
+++ b/flink-core/src/main/java/org/apache/flink/core/memory/ByteArrayOutputStreamWithPos.java
@@ -18,6 +18,7 @@
 
 package org.apache.flink.core.memory;
 
+import org.apache.flink.annotation.Internal;
 import org.apache.flink.util.Preconditions;
 
 import java.io.IOException;
@@ -27,6 +28,7 @@ import java.util.Arrays;
 /**
  * Un-synchronized stream similar to Java's ByteArrayOutputStream that also exposes the current position.
  */
+@Internal
 public class ByteArrayOutputStreamWithPos extends OutputStream {
 
 	protected byte[] buffer;

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/util/AbstractCloseableRegistry.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/util/AbstractCloseableRegistry.java b/flink-core/src/main/java/org/apache/flink/util/AbstractCloseableRegistry.java
index 85af982..e165d97 100644
--- a/flink-core/src/main/java/org/apache/flink/util/AbstractCloseableRegistry.java
+++ b/flink-core/src/main/java/org/apache/flink/util/AbstractCloseableRegistry.java
@@ -18,6 +18,8 @@
 
 package org.apache.flink.util;
 
+import org.apache.flink.annotation.Internal;
+
 import java.io.Closeable;
 import java.io.IOException;
 import java.util.Map;
@@ -33,6 +35,7 @@ import java.util.Map;
  * @param <C> Type of the closeable this registers
  * @param <T> Type for potential meta data associated with the registering closeables
  */
+@Internal
 public abstract class AbstractCloseableRegistry<C extends Closeable, T> implements Closeable {
 
 	protected final Map<Closeable, T> closeableToRef;

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/util/CollectionUtil.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/util/CollectionUtil.java b/flink-core/src/main/java/org/apache/flink/util/CollectionUtil.java
index cd5c2e5..bd6847a 100644
--- a/flink-core/src/main/java/org/apache/flink/util/CollectionUtil.java
+++ b/flink-core/src/main/java/org/apache/flink/util/CollectionUtil.java
@@ -18,9 +18,12 @@
 
 package org.apache.flink.util;
 
+import org.apache.flink.annotation.Internal;
+
 import java.util.Collection;
 import java.util.Map;
 
+@Internal
 public final class CollectionUtil {
 
 	private CollectionUtil() {

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/util/FutureUtil.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/util/FutureUtil.java b/flink-core/src/main/java/org/apache/flink/util/FutureUtil.java
index 62d836b..5ecf344 100644
--- a/flink-core/src/main/java/org/apache/flink/util/FutureUtil.java
+++ b/flink-core/src/main/java/org/apache/flink/util/FutureUtil.java
@@ -18,9 +18,12 @@
 
 package org.apache.flink.util;
 
+import org.apache.flink.annotation.Internal;
+
 import java.util.concurrent.ExecutionException;
 import java.util.concurrent.RunnableFuture;
 
+@Internal
 public class FutureUtil {
 
 	private FutureUtil() {
@@ -39,4 +42,4 @@ public class FutureUtil {
 
 		return future.get();
 	}
-}
\ No newline at end of file
+}

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/util/Migration.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/util/Migration.java b/flink-core/src/main/java/org/apache/flink/util/Migration.java
index 4bd9e39..a82488d 100644
--- a/flink-core/src/main/java/org/apache/flink/util/Migration.java
+++ b/flink-core/src/main/java/org/apache/flink/util/Migration.java
@@ -18,8 +18,11 @@
 
 package org.apache.flink.util;
 
+import org.apache.flink.annotation.Internal;
+
 /**
  * Tagging interface for migration related classes.
  */
+@Internal
 public interface Migration {
 }

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/util/WrappingProxy.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/util/WrappingProxy.java b/flink-core/src/main/java/org/apache/flink/util/WrappingProxy.java
index 82fcf04..213feee 100644
--- a/flink-core/src/main/java/org/apache/flink/util/WrappingProxy.java
+++ b/flink-core/src/main/java/org/apache/flink/util/WrappingProxy.java
@@ -18,6 +18,9 @@
 
 package org.apache.flink.util;
 
+import org.apache.flink.annotation.Internal;
+
+@Internal
 public interface WrappingProxy<T> {
 
 	T getWrappedDelegate();

http://git-wip-us.apache.org/repos/asf/flink/blob/7cdbd3c6/flink-core/src/main/java/org/apache/flink/util/WrappingProxyUtil.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/util/WrappingProxyUtil.java b/flink-core/src/main/java/org/apache/flink/util/WrappingProxyUtil.java
index 0f62abd..6a79913 100644
--- a/flink-core/src/main/java/org/apache/flink/util/WrappingProxyUtil.java
+++ b/flink-core/src/main/java/org/apache/flink/util/WrappingProxyUtil.java
@@ -18,6 +18,9 @@
 
 package org.apache.flink.util;
 
+import org.apache.flink.annotation.Internal;
+
+@Internal
 public final class WrappingProxyUtil {
 
 	private WrappingProxyUtil() {
@@ -30,4 +33,4 @@ public final class WrappingProxyUtil {
 		}
 		return object;
 	}
-}
\ No newline at end of file
+}


[9/9] flink git commit: [FLINK-4813] [test-utils] Make the hadoop-minikdc dependency optional

Posted by se...@apache.org.
[FLINK-4813] [test-utils] Make the hadoop-minikdc dependency optional

With this change, any project using flink-test-utils which also requires
SecureTestEnvironment must add a dependency to hadoop-minikdc itself, e.g. in
pom.xml:

   ...
   <dependencies>
     <dependency>
       <groupId>org.apache.hadoop</groupId>
       <artifactId>hadoop-minikdc</artifactId>
       <version>${minikdc.version}</version>
       <scope>compile</scope>
     </dependency>
   ...
   </dependencies>
   ...

   <build>
     <plugins>
       <!--
         https://issues.apache.org/jira/browse/DIRSHARED-134
         Required to pull the Mini-KDC transitive dependency
       -->
       <plugin>
       <groupId>org.apache.felix</groupId>
       <artifactId>maven-bundle-plugin</artifactId>
       <version>3.0.1</version>
       <inherited>true</inherited>
       <extensions>true</extensions>
     </plugin>
   ...

This closes #3322


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

Branch: refs/heads/release-1.2
Commit: 2c08f7548ad8388ab081f82cd6f6c916c8900ee0
Parents: a082335
Author: Nico Kruber <ni...@data-artisans.com>
Authored: Wed Feb 15 14:24:32 2017 +0100
Committer: Stephan Ewen <se...@apache.org>
Committed: Mon Feb 20 02:07:55 2017 +0100

----------------------------------------------------------------------
 .../flink-test-utils/pom.xml                    | 23 +++++++++++++
 .../flink/test/util/SecureTestEnvironment.java  | 34 ++++++++++++++++++--
 pom.xml                                         | 10 ------
 3 files changed, 55 insertions(+), 12 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/flink/blob/2c08f754/flink-test-utils-parent/flink-test-utils/pom.xml
----------------------------------------------------------------------
diff --git a/flink-test-utils-parent/flink-test-utils/pom.xml b/flink-test-utils-parent/flink-test-utils/pom.xml
index 875a2bf..da10cda 100644
--- a/flink-test-utils-parent/flink-test-utils/pom.xml
+++ b/flink-test-utils-parent/flink-test-utils/pom.xml
@@ -83,8 +83,31 @@ under the License.
 			<artifactId>hadoop-minikdc</artifactId>
 			<version>${minikdc.version}</version>
 			<scope>compile</scope>
+			<!-- Since this requires the maven-bundle-plugin to be present,
+			make it optional.
+			The only class using this dependency is SecureTestEnvironment and if
+			a project is using it, it must now add the hadoop-minikdc dependency
+			on its own as well as include maven-bundle-plugin.
+			-->
+			<optional>true</optional>
 		</dependency>
 
 	</dependencies>
 
+	<build>
+		<plugins>
+			<!--
+            https://issues.apache.org/jira/browse/DIRSHARED-134
+            Required to pull the Mini-KDC transitive dependency
+            -->
+			<plugin>
+				<groupId>org.apache.felix</groupId>
+				<artifactId>maven-bundle-plugin</artifactId>
+				<version>3.0.1</version>
+				<inherited>true</inherited>
+				<extensions>true</extensions>
+			</plugin>
+		</plugins>
+	</build>
+
 </project>

http://git-wip-us.apache.org/repos/asf/flink/blob/2c08f754/flink-test-utils-parent/flink-test-utils/src/main/java/org/apache/flink/test/util/SecureTestEnvironment.java
----------------------------------------------------------------------
diff --git a/flink-test-utils-parent/flink-test-utils/src/main/java/org/apache/flink/test/util/SecureTestEnvironment.java b/flink-test-utils-parent/flink-test-utils/src/main/java/org/apache/flink/test/util/SecureTestEnvironment.java
index 10450c3..febd074 100644
--- a/flink-test-utils-parent/flink-test-utils/src/main/java/org/apache/flink/test/util/SecureTestEnvironment.java
+++ b/flink-test-utils-parent/flink-test-utils/src/main/java/org/apache/flink/test/util/SecureTestEnvironment.java
@@ -37,9 +37,39 @@ import java.util.Properties;
 /**
  * Helper {@link SecureTestEnvironment} to handle MiniKDC lifecycle.
  * This class can be used to start/stop MiniKDC and create secure configurations for MiniDFSCluster
- * and MiniYarn
+ * and MiniYarn.
+ *
+ * If you use this class in your project, please make sure to add a dependency to
+ * <tt>hadoop-minikdc</tt>, e.g. in your <tt>pom.xml</tt>:
+ * <pre>{@code
+ * ...
+ * <dependencies>
+ *   <dependency>
+ *     <groupId>org.apache.hadoop</groupId>
+ *     <artifactId>hadoop-minikdc</artifactId>
+ *     <version>${minikdc.version}</version>
+ *     <scope>compile</scope>
+ *   </dependency>
+ * ...
+ * </dependencies>
+ * ...
+ *
+ * <build>
+ *   <plugins>
+ *     <!--
+ *       https://issues.apache.org/jira/browse/DIRSHARED-134
+ *       Required to pull the Mini-KDC transitive dependency
+ *     -->
+ *     <plugin>
+ *     <groupId>org.apache.felix</groupId>
+ *     <artifactId>maven-bundle-plugin</artifactId>
+ *     <version>3.0.1</version>
+ *     <inherited>true</inherited>
+ *     <extensions>true</extensions>
+ *   </plugin>
+ * ...
+ * }</pre>
  */
-
 public class SecureTestEnvironment {
 
 	protected static final Logger LOG = LoggerFactory.getLogger(SecureTestEnvironment.class);

http://git-wip-us.apache.org/repos/asf/flink/blob/2c08f754/pom.xml
----------------------------------------------------------------------
diff --git a/pom.xml b/pom.xml
index a1bca92..b3d6483 100644
--- a/pom.xml
+++ b/pom.xml
@@ -1113,16 +1113,6 @@ under the License.
 				</executions>
 			</plugin>
 
-			<!-- Pull bundled transitive dependencies (i.e. Mini-KDC).
-			See https://issues.apache.org/jira/browse/DIRSHARED-134 -->
-			<plugin>
-				<groupId>org.apache.felix</groupId>
-				<artifactId>maven-bundle-plugin</artifactId>
-				<version>3.0.1</version>
-				<inherited>true</inherited>
-				<extensions>true</extensions>
-			</plugin>
-
 		</plugins>
 
 		<!-- Plugin configurations for plugins activated in sub-projects --> 


[7/9] flink git commit: [hotfix] [core] Add missing @PublicEvolving annotations to classes in flink-core.

Posted by se...@apache.org.
[hotfix] [core] Add missing @PublicEvolving annotations to classes in flink-core.


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

Branch: refs/heads/release-1.2
Commit: 719094411de8d5442d7220f2a5404a4768ef079d
Parents: 2c08f75
Author: Fabian Hueske <fh...@apache.org>
Authored: Wed Feb 15 15:25:01 2017 +0100
Committer: Stephan Ewen <se...@apache.org>
Committed: Mon Feb 20 02:07:55 2017 +0100

----------------------------------------------------------------------
 .../src/main/java/org/apache/flink/api/common/Archiveable.java  | 3 +++
 .../java/org/apache/flink/configuration/SecurityOptions.java    | 3 +++
 .../java/org/apache/flink/core/io/VersionMismatchException.java | 5 ++++-
 .../src/main/java/org/apache/flink/core/io/Versioned.java       | 3 +++
 .../org/apache/flink/core/io/VersionedIOReadableWritable.java   | 4 +++-
 .../apache/flink/migration/util/MigrationInstantiationUtil.java | 2 ++
 .../java/org/apache/flink/migration/util/SerializedValue.java   | 2 ++
 7 files changed, 20 insertions(+), 2 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/flink/blob/71909441/flink-core/src/main/java/org/apache/flink/api/common/Archiveable.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/api/common/Archiveable.java b/flink-core/src/main/java/org/apache/flink/api/common/Archiveable.java
index 09a3a0c..69e050d 100644
--- a/flink-core/src/main/java/org/apache/flink/api/common/Archiveable.java
+++ b/flink-core/src/main/java/org/apache/flink/api/common/Archiveable.java
@@ -17,8 +17,11 @@
  */
 package org.apache.flink.api.common;
 
+import org.apache.flink.annotation.PublicEvolving;
+
 import java.io.Serializable;
 
+@PublicEvolving
 public interface Archiveable<T extends Serializable> {
 	T archive();
 }

http://git-wip-us.apache.org/repos/asf/flink/blob/71909441/flink-core/src/main/java/org/apache/flink/configuration/SecurityOptions.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/configuration/SecurityOptions.java b/flink-core/src/main/java/org/apache/flink/configuration/SecurityOptions.java
index 67d101d..95cf0c7 100644
--- a/flink-core/src/main/java/org/apache/flink/configuration/SecurityOptions.java
+++ b/flink-core/src/main/java/org/apache/flink/configuration/SecurityOptions.java
@@ -18,11 +18,14 @@
 
 package org.apache.flink.configuration;
 
+import org.apache.flink.annotation.PublicEvolving;
+
 import static org.apache.flink.configuration.ConfigOptions.key;
 
 /**
  * The set of configuration options relating to security.
  */
+@PublicEvolving
 public class SecurityOptions {
 
 	// ------------------------------------------------------------------------

http://git-wip-us.apache.org/repos/asf/flink/blob/71909441/flink-core/src/main/java/org/apache/flink/core/io/VersionMismatchException.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/core/io/VersionMismatchException.java b/flink-core/src/main/java/org/apache/flink/core/io/VersionMismatchException.java
index 3ff88e9..92fd4f4 100644
--- a/flink-core/src/main/java/org/apache/flink/core/io/VersionMismatchException.java
+++ b/flink-core/src/main/java/org/apache/flink/core/io/VersionMismatchException.java
@@ -18,11 +18,14 @@
 
 package org.apache.flink.core.io;
 
+import org.apache.flink.annotation.PublicEvolving;
+
 import java.io.IOException;
 
 /**
  * This exception signals that incompatible versions have been found during serialization.
  */
+@PublicEvolving
 public class VersionMismatchException extends IOException {
 
 	private static final long serialVersionUID = 7024258967585372438L;
@@ -41,4 +44,4 @@ public class VersionMismatchException extends IOException {
 	public VersionMismatchException(Throwable cause) {
 		super(cause);
 	}
-}
\ No newline at end of file
+}

http://git-wip-us.apache.org/repos/asf/flink/blob/71909441/flink-core/src/main/java/org/apache/flink/core/io/Versioned.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/core/io/Versioned.java b/flink-core/src/main/java/org/apache/flink/core/io/Versioned.java
index b36d5e8..786bf73 100644
--- a/flink-core/src/main/java/org/apache/flink/core/io/Versioned.java
+++ b/flink-core/src/main/java/org/apache/flink/core/io/Versioned.java
@@ -18,10 +18,13 @@
 
 package org.apache.flink.core.io;
 
+import org.apache.flink.annotation.PublicEvolving;
+
 /**
  * This interface is implemented by classes that provide a version number. Versions numbers can be used to differentiate
  * between evolving classes.
  */
+@PublicEvolving
 public interface Versioned {
 
 	/**

http://git-wip-us.apache.org/repos/asf/flink/blob/71909441/flink-core/src/main/java/org/apache/flink/core/io/VersionedIOReadableWritable.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/core/io/VersionedIOReadableWritable.java b/flink-core/src/main/java/org/apache/flink/core/io/VersionedIOReadableWritable.java
index 94c2722..31d570c 100644
--- a/flink-core/src/main/java/org/apache/flink/core/io/VersionedIOReadableWritable.java
+++ b/flink-core/src/main/java/org/apache/flink/core/io/VersionedIOReadableWritable.java
@@ -18,6 +18,7 @@
 
 package org.apache.flink.core.io;
 
+import org.apache.flink.annotation.PublicEvolving;
 import org.apache.flink.core.memory.DataInputView;
 import org.apache.flink.core.memory.DataOutputView;
 
@@ -28,6 +29,7 @@ import java.io.IOException;
  * versions. Concrete subclasses should typically override the {@link #write(DataOutputView)} and
  * {@link #read(DataInputView)}, thereby calling super to ensure version checking.
  */
+@PublicEvolving
 public abstract class VersionedIOReadableWritable implements IOReadableWritable, Versioned {
 
 	@Override
@@ -67,4 +69,4 @@ public abstract class VersionedIOReadableWritable implements IOReadableWritable,
 	public boolean isCompatibleVersion(int version) {
 		return getVersion() == version;
 	}
-}
\ No newline at end of file
+}

http://git-wip-us.apache.org/repos/asf/flink/blob/71909441/flink-core/src/main/java/org/apache/flink/migration/util/MigrationInstantiationUtil.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/migration/util/MigrationInstantiationUtil.java b/flink-core/src/main/java/org/apache/flink/migration/util/MigrationInstantiationUtil.java
index 2e8b144..43051ed 100644
--- a/flink-core/src/main/java/org/apache/flink/migration/util/MigrationInstantiationUtil.java
+++ b/flink-core/src/main/java/org/apache/flink/migration/util/MigrationInstantiationUtil.java
@@ -18,6 +18,7 @@
 
 package org.apache.flink.migration.util;
 
+import org.apache.flink.annotation.PublicEvolving;
 import org.apache.flink.util.InstantiationUtil;
 
 import java.io.ByteArrayInputStream;
@@ -29,6 +30,7 @@ import java.io.ObjectStreamClass;
 /**
  * Utility class to deserialize legacy classes for migration.
  */
+@PublicEvolving
 public final class MigrationInstantiationUtil {
 
 	public static class ClassLoaderObjectInputStream extends InstantiationUtil.ClassLoaderObjectInputStream {

http://git-wip-us.apache.org/repos/asf/flink/blob/71909441/flink-core/src/main/java/org/apache/flink/migration/util/SerializedValue.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/migration/util/SerializedValue.java b/flink-core/src/main/java/org/apache/flink/migration/util/SerializedValue.java
index aab68c9..bd40b83 100644
--- a/flink-core/src/main/java/org/apache/flink/migration/util/SerializedValue.java
+++ b/flink-core/src/main/java/org/apache/flink/migration/util/SerializedValue.java
@@ -18,6 +18,7 @@
 
 package org.apache.flink.migration.util;
 
+import org.apache.flink.annotation.PublicEvolving;
 import org.apache.flink.util.InstantiationUtil;
 
 import java.io.IOException;
@@ -36,6 +37,7 @@ import java.util.Arrays;
  * @param <T> The type of the value held.
  */
 @Deprecated
+@PublicEvolving
 public class SerializedValue<T> implements java.io.Serializable {
 
 	private static final long serialVersionUID = -3564011643393683761L;


[3/9] flink git commit: [FLINK-5640] [build] Configure the explicit Unit Test file suffix

Posted by se...@apache.org.
[FLINK-5640] [build] Configure the explicit Unit Test file suffix

This closes #3211


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

Branch: refs/heads/release-1.2
Commit: efa5e9d4b83277a34ac20d1a258727b7008a8ec1
Parents: fc5a20a
Author: shijinkui <sh...@huawei.com>
Authored: Fri Feb 17 00:24:20 2017 +0800
Committer: Stephan Ewen <se...@apache.org>
Committed: Mon Feb 20 02:07:54 2017 +0100

----------------------------------------------------------------------
 pom.xml | 7 +++++++
 1 file changed, 7 insertions(+)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/flink/blob/efa5e9d4/pom.xml
----------------------------------------------------------------------
diff --git a/pom.xml b/pom.xml
index 50a25f2..a1bca92 100644
--- a/pom.xml
+++ b/pom.xml
@@ -958,6 +958,8 @@ under the License.
 					<compilerArgument>-Xlint:all</compilerArgument>
 				</configuration>
 			</plugin>
+
+			<!--surefire for unit tests and integration tests-->
 			<plugin>
 				<groupId>org.apache.maven.plugins</groupId>
 				<artifactId>maven-surefire-plugin</artifactId>
@@ -973,6 +975,7 @@ under the License.
 					<argLine>-Xms256m -Xmx800m -Dmvn.forkNumber=${surefire.forkNumber} -XX:-UseGCOverheadLimit</argLine>
 				</configuration>
 				<executions>
+					<!--execute all the unit tests-->
 					<execution>
 						<id>default-test</id>
 						<phase>test</phase>
@@ -980,12 +983,16 @@ under the License.
 							<goal>test</goal>
 						</goals>
 						<configuration>
+							<includes>
+								<include>**/*Test.*</include>
+							</includes>
 							<excludes>
 								<exclude>**/*ITCase.*</exclude>
 								<exclude>${flink-fast-tests-pattern}</exclude>
 							</excludes>
 						</configuration>
 					</execution>
+					<!--execute all the integration tests-->
 					<execution>
 						<id>integration-tests</id>
 						<phase>integration-test</phase>