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 2016/08/30 13:28:25 UTC

[1/2] flink git commit: [FLINK-4271] [DataStream API] Extends JavaDocs for 'apply(...)' and 'with(...)' functions.

Repository: flink
Updated Branches:
  refs/heads/master 96b6359fc -> 78d9ae9ba


[FLINK-4271] [DataStream API] Extends JavaDocs for 'apply(...)' and 'with(...)' functions.


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

Branch: refs/heads/master
Commit: 78d9ae9ba3ddda4b84d335b07ece7c04b18d4751
Parents: 2a20229
Author: Stephan Ewen <se...@apache.org>
Authored: Tue Aug 30 12:28:07 2016 +0200
Committer: Stephan Ewen <se...@apache.org>
Committed: Tue Aug 30 15:27:53 2016 +0200

----------------------------------------------------------------------
 .../api/datastream/CoGroupedStreams.java        | 32 +++++++----
 .../streaming/api/datastream/JoinedStreams.java | 56 +++++++++++++-------
 2 files changed, 59 insertions(+), 29 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/flink/blob/78d9ae9b/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/datastream/CoGroupedStreams.java
----------------------------------------------------------------------
diff --git a/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/datastream/CoGroupedStreams.java b/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/datastream/CoGroupedStreams.java
index 0bab4cf..62d032d 100644
--- a/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/datastream/CoGroupedStreams.java
+++ b/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/datastream/CoGroupedStreams.java
@@ -224,6 +224,10 @@ public class CoGroupedStreams<T1, T2> {
 		/**
 		 * Completes the co-group operation with the user function that is executed
 		 * for windowed groups.
+		 * 
+		 * <p>Note: This method's return type does not support setting an operator-specific parallelism.
+		 * Due to binary backwards compatibility, this cannot be altered. Use the {@link #with(CoGroupFunction)}
+		 * method to set an operator-specific parallelism.
 		 */
 		public <T> DataStream<T> apply(CoGroupFunction<T1, T2, T> function) {
 
@@ -244,12 +248,13 @@ public class CoGroupedStreams<T1, T2> {
 		 * Completes the co-group operation with the user function that is executed
 		 * for windowed groups.
 		 *
-		 * <p>
-		 *     Note: This is a temporary workaround while the {@link #apply(CoGroupFunction)} method has the wrong return type.
-		 * </p>
-		 * @deprecated This method will be replaced by {@link #apply(CoGroupFunction)} in Flink 2.0.
-		 * So use the {@link #apply(CoGroupFunction)} in the future.
-         */
+		 * <p><b>Note:</b> This is a temporary workaround while the {@link #apply(CoGroupFunction)}
+		 * method has the wrong return type and hence does not allow one to set an operator-specific
+		 * parallelism
+		 * 
+		 * @deprecated This method will be removed once the {@link #apply(CoGroupFunction)} method is fixed
+		 *             in the next major version of Flink (2.0).
+		 */
 		@PublicEvolving
 		@Deprecated
 		public <T> SingleOutputStreamOperator<T> with(CoGroupFunction<T1, T2, T> function) {
@@ -259,6 +264,10 @@ public class CoGroupedStreams<T1, T2> {
 		/**
 		 * Completes the co-group operation with the user function that is executed
 		 * for windowed groups.
+		 * 
+		 * <p>Note: This method's return type does not support setting an operator-specific parallelism.
+		 * Due to binary backwards compatibility, this cannot be altered. Use the
+		 * {@link #with(CoGroupFunction, TypeInformation)} method to set an operator-specific parallelism.
 		 */
 		public <T> DataStream<T> apply(CoGroupFunction<T1, T2, T> function, TypeInformation<T> resultType) {
 			//clean the closure
@@ -297,11 +306,12 @@ public class CoGroupedStreams<T1, T2> {
 		 * Completes the co-group operation with the user function that is executed
 		 * for windowed groups.
 		 *
-		 * <p>
-		 *     Note: This is a temporary workaround while the {@link #apply(CoGroupFunction, TypeInformation)} method has the wrong return type.
-		 * </p>
-		 * @deprecated This method will be replaced by {@link #apply(CoGroupFunction, TypeInformation)} in Flink 2.0.
-		 * So use the {@link #apply(CoGroupFunction, TypeInformation)} in the future.
+		 * <p><b>Note:</b> This is a temporary workaround while the {@link #apply(CoGroupFunction, TypeInformation)}
+		 * method has the wrong return type and hence does not allow one to set an operator-specific
+		 * parallelism
+		 *
+		 * @deprecated This method will be removed once the {@link #apply(CoGroupFunction, TypeInformation)}
+		 *             method is fixed in the next major version of Flink (2.0).
 		 */
 		@PublicEvolving
 		@Deprecated

http://git-wip-us.apache.org/repos/asf/flink/blob/78d9ae9b/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/datastream/JoinedStreams.java
----------------------------------------------------------------------
diff --git a/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/datastream/JoinedStreams.java b/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/datastream/JoinedStreams.java
index 8f8fc67..c005310 100644
--- a/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/datastream/JoinedStreams.java
+++ b/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/datastream/JoinedStreams.java
@@ -219,6 +219,10 @@ public class JoinedStreams<T1, T2> {
 		/**
 		 * Completes the join operation with the user function that is executed
 		 * for each combination of elements with the same key in a window.
+		 * 
+		 * <p>Note: This method's return type does not support setting an operator-specific parallelism.
+		 * Due to binary backwards compatibility, this cannot be altered. Use the {@link #with(JoinFunction)}
+		 * method to set an operator-specific parallelism.
 		 */
 		public <T> DataStream<T> apply(JoinFunction<T1, T2, T> function) {
 			TypeInformation<T> resultType = TypeExtractor.getBinaryOperatorReturnType(
@@ -238,11 +242,12 @@ public class JoinedStreams<T1, T2> {
 		 * Completes the join operation with the user function that is executed
 		 * for each combination of elements with the same key in a window.
 		 *
-		 * <p>
-		 *     Note: This is a temporary workaround while the {@link #apply(JoinFunction)} method has the wrong return type.
-		 * </p>
-		 * @deprecated This method will be replaced by {@link #apply(JoinFunction)} in Flink 2.0.
-		 * So use the {@link #apply(JoinFunction)} in the future.
+		 * <p><b>Note:</b> This is a temporary workaround while the {@link #apply(JoinFunction)}
+		 * method has the wrong return type and hence does not allow one to set an operator-specific
+		 * parallelism
+		 * 
+		 * @deprecated This method will be removed once the {@link #apply(JoinFunction)} method is fixed
+		 *             in the next major version of Flink (2.0).
 		 */
 		@PublicEvolving
 		@Deprecated
@@ -253,6 +258,10 @@ public class JoinedStreams<T1, T2> {
 		/**
 		 * Completes the join operation with the user function that is executed
 		 * for each combination of elements with the same key in a window.
+		 * 
+		 * <p>Note: This method's return type does not support setting an operator-specific parallelism.
+		 * Due to binary backwards compatibility, this cannot be altered. Use the
+		 * {@link #with(JoinFunction, TypeInformation)}, method to set an operator-specific parallelism.
 		 */
 		public <T> DataStream<T> apply(FlatJoinFunction<T1, T2, T> function, TypeInformation<T> resultType) {
 			//clean the closure
@@ -273,9 +282,10 @@ public class JoinedStreams<T1, T2> {
 		 * Completes the join operation with the user function that is executed
 		 * for each combination of elements with the same key in a window.
 		 *
-		 * <p>
-		 *     Note: This is a temporary workaround while the {@link #apply(FlatJoinFunction, TypeInformation)} method has the wrong return type.
-		 * </p>
+		 * <p><b>Note:</b> This is a temporary workaround while the {@link #apply(JoinFunction, TypeInformation)}
+		 * method has the wrong return type and hence does not allow one to set an operator-specific
+		 * parallelism
+		 *
 		 * @deprecated This method will be replaced by {@link #apply(FlatJoinFunction, TypeInformation)} in Flink 2.0.
 		 * So use the {@link #apply(FlatJoinFunction, TypeInformation)} in the future.
 		 */
@@ -288,6 +298,10 @@ public class JoinedStreams<T1, T2> {
 		/**
 		 * Completes the join operation with the user function that is executed
 		 * for each combination of elements with the same key in a window.
+		 * 
+		 * <p>Note: This method's return type does not support setting an operator-specific parallelism.
+		 * Due to binary backwards compatibility, this cannot be altered. Use the
+		 * {@link #with(FlatJoinFunction)}, method to set an operator-specific parallelism.
 		 */
 		public <T> DataStream<T> apply(FlatJoinFunction<T1, T2, T> function) {
 			TypeInformation<T> resultType = TypeExtractor.getBinaryOperatorReturnType(
@@ -307,11 +321,12 @@ public class JoinedStreams<T1, T2> {
 		 * Completes the join operation with the user function that is executed
 		 * for each combination of elements with the same key in a window.
 		 *
-		 * <p>
-		 *     Note: This is a temporary workaround while the {@link #apply(FlatJoinFunction)} method has the wrong return type.
-		 * </p>
-		 * @deprecated This method will be replaced by {@link #apply(FlatJoinFunction)} in Flink 2.0.
-		 * So use the {@link #apply(FlatJoinFunction)} in the future.
+		 * <p><b>Note:</b> This is a temporary workaround while the {@link #apply(FlatJoinFunction)}
+		 * method has the wrong return type and hence does not allow one to set an operator-specific
+		 * parallelism.
+		 * 
+		 * @deprecated This method will be removed once the {@link #apply(FlatJoinFunction)}
+		 *             method is fixed in the next major version of Flink (2.0).
 		 */
 		@PublicEvolving
 		@Deprecated
@@ -322,6 +337,10 @@ public class JoinedStreams<T1, T2> {
 		/**
 		 * Completes the join operation with the user function that is executed
 		 * for each combination of elements with the same key in a window.
+		 * 
+		 * <p>Note: This method's return type does not support setting an operator-specific parallelism.
+		 * Due to binary backwards compatibility, this cannot be altered. Use the
+		 * {@link #with(JoinFunction, TypeInformation)}, method to set an operator-specific parallelism.
 		 */
 		public <T> DataStream<T> apply(JoinFunction<T1, T2, T> function, TypeInformation<T> resultType) {
 			//clean the closure
@@ -341,11 +360,12 @@ public class JoinedStreams<T1, T2> {
 		 * Completes the join operation with the user function that is executed
 		 * for each combination of elements with the same key in a window.
 		 *
-		 * <p>
-		 *     Note: This is a temporary workaround while the {@link #apply(JoinFunction, TypeInformation)} method has the wrong return type.
-		 * </p>
-		 * @deprecated This method will be replaced by {@link #apply(JoinFunction, TypeInformation)} in Flink 2.0.
-		 * So use the {@link #apply(JoinFunction, TypeInformation)} in the future.
+		 * <p><b>Note:</b> This is a temporary workaround while the {@link #apply(FlatJoinFunction, TypeInformation)}
+		 * method has the wrong return type and hence does not allow one to set an operator-specific
+		 * parallelism
+		 * 
+		 * @deprecated This method will be removed once the {@link #apply(JoinFunction, TypeInformation)}
+		 *             method is fixed in the next major version of Flink (2.0).
 		 */
 		@PublicEvolving
 		@Deprecated


[2/2] flink git commit: [FLINK-4271] [DataStream API] Enable CoGroupedStreams and JoinedStreams to set parallelism.

Posted by se...@apache.org.
[FLINK-4271] [DataStream API] Enable CoGroupedStreams and JoinedStreams to set parallelism.

This closes #2305


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

Branch: refs/heads/master
Commit: 2a20229b56997ff1249a879acfb2bcca4323b0a1
Parents: 96b6359
Author: Jark Wu <wu...@alibaba-inc.com>
Authored: Thu Jul 28 14:32:13 2016 +0800
Committer: Stephan Ewen <se...@apache.org>
Committed: Tue Aug 30 15:27:53 2016 +0200

----------------------------------------------------------------------
 .../api/datastream/CoGroupedStreams.java        | 34 ++++++++++
 .../streaming/api/datastream/JoinedStreams.java | 65 ++++++++++++++++++++
 2 files changed, 99 insertions(+)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/flink/blob/2a20229b/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/datastream/CoGroupedStreams.java
----------------------------------------------------------------------
diff --git a/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/datastream/CoGroupedStreams.java b/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/datastream/CoGroupedStreams.java
index a9a64af..0bab4cf 100644
--- a/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/datastream/CoGroupedStreams.java
+++ b/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/datastream/CoGroupedStreams.java
@@ -243,6 +243,22 @@ public class CoGroupedStreams<T1, T2> {
 		/**
 		 * Completes the co-group operation with the user function that is executed
 		 * for windowed groups.
+		 *
+		 * <p>
+		 *     Note: This is a temporary workaround while the {@link #apply(CoGroupFunction)} method has the wrong return type.
+		 * </p>
+		 * @deprecated This method will be replaced by {@link #apply(CoGroupFunction)} in Flink 2.0.
+		 * So use the {@link #apply(CoGroupFunction)} in the future.
+         */
+		@PublicEvolving
+		@Deprecated
+		public <T> SingleOutputStreamOperator<T> with(CoGroupFunction<T1, T2, T> function) {
+			return (SingleOutputStreamOperator<T>) apply(function);
+		}
+
+		/**
+		 * Completes the co-group operation with the user function that is executed
+		 * for windowed groups.
 		 */
 		public <T> DataStream<T> apply(CoGroupFunction<T1, T2, T> function, TypeInformation<T> resultType) {
 			//clean the closure
@@ -253,9 +269,11 @@ public class CoGroupedStreams<T1, T2> {
 			
 			DataStream<TaggedUnion<T1, T2>> taggedInput1 = input1
 					.map(new Input1Tagger<T1, T2>())
+					.setParallelism(input1.getParallelism())
 					.returns(unionType);
 			DataStream<TaggedUnion<T1, T2>> taggedInput2 = input2
 					.map(new Input2Tagger<T1, T2>())
+					.setParallelism(input2.getParallelism())
 					.returns(unionType);
 
 			DataStream<TaggedUnion<T1, T2>> unionStream = taggedInput1.union(taggedInput2);
@@ -274,6 +292,22 @@ public class CoGroupedStreams<T1, T2> {
 
 			return windowOp.apply(new CoGroupWindowFunction<T1, T2, T, KEY, W>(function), resultType);
 		}
+
+		/**
+		 * Completes the co-group operation with the user function that is executed
+		 * for windowed groups.
+		 *
+		 * <p>
+		 *     Note: This is a temporary workaround while the {@link #apply(CoGroupFunction, TypeInformation)} method has the wrong return type.
+		 * </p>
+		 * @deprecated This method will be replaced by {@link #apply(CoGroupFunction, TypeInformation)} in Flink 2.0.
+		 * So use the {@link #apply(CoGroupFunction, TypeInformation)} in the future.
+		 */
+		@PublicEvolving
+		@Deprecated
+		public <T> SingleOutputStreamOperator<T> with(CoGroupFunction<T1, T2, T> function, TypeInformation<T> resultType) {
+			return (SingleOutputStreamOperator<T>) apply(function, resultType);
+		}
 	}
 
 	// ------------------------------------------------------------------------

http://git-wip-us.apache.org/repos/asf/flink/blob/2a20229b/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/datastream/JoinedStreams.java
----------------------------------------------------------------------
diff --git a/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/datastream/JoinedStreams.java b/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/datastream/JoinedStreams.java
index 86c6226..8f8fc67 100644
--- a/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/datastream/JoinedStreams.java
+++ b/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/datastream/JoinedStreams.java
@@ -237,6 +237,22 @@ public class JoinedStreams<T1, T2> {
 		/**
 		 * Completes the join operation with the user function that is executed
 		 * for each combination of elements with the same key in a window.
+		 *
+		 * <p>
+		 *     Note: This is a temporary workaround while the {@link #apply(JoinFunction)} method has the wrong return type.
+		 * </p>
+		 * @deprecated This method will be replaced by {@link #apply(JoinFunction)} in Flink 2.0.
+		 * So use the {@link #apply(JoinFunction)} in the future.
+		 */
+		@PublicEvolving
+		@Deprecated
+		public <T> SingleOutputStreamOperator<T> with(JoinFunction<T1, T2, T> function) {
+			return (SingleOutputStreamOperator<T>) apply(function);
+		}
+
+		/**
+		 * Completes the join operation with the user function that is executed
+		 * for each combination of elements with the same key in a window.
 		 */
 		public <T> DataStream<T> apply(FlatJoinFunction<T1, T2, T> function, TypeInformation<T> resultType) {
 			//clean the closure
@@ -252,6 +268,23 @@ public class JoinedStreams<T1, T2> {
 
 		}
 
+
+		/**
+		 * Completes the join operation with the user function that is executed
+		 * for each combination of elements with the same key in a window.
+		 *
+		 * <p>
+		 *     Note: This is a temporary workaround while the {@link #apply(FlatJoinFunction, TypeInformation)} method has the wrong return type.
+		 * </p>
+		 * @deprecated This method will be replaced by {@link #apply(FlatJoinFunction, TypeInformation)} in Flink 2.0.
+		 * So use the {@link #apply(FlatJoinFunction, TypeInformation)} in the future.
+		 */
+		@PublicEvolving
+		@Deprecated
+		public <T> SingleOutputStreamOperator<T> with(FlatJoinFunction<T1, T2, T> function, TypeInformation<T> resultType) {
+			return (SingleOutputStreamOperator<T>) apply(function, resultType);
+		}
+
 		/**
 		 * Completes the join operation with the user function that is executed
 		 * for each combination of elements with the same key in a window.
@@ -273,6 +306,22 @@ public class JoinedStreams<T1, T2> {
 		/**
 		 * Completes the join operation with the user function that is executed
 		 * for each combination of elements with the same key in a window.
+		 *
+		 * <p>
+		 *     Note: This is a temporary workaround while the {@link #apply(FlatJoinFunction)} method has the wrong return type.
+		 * </p>
+		 * @deprecated This method will be replaced by {@link #apply(FlatJoinFunction)} in Flink 2.0.
+		 * So use the {@link #apply(FlatJoinFunction)} in the future.
+		 */
+		@PublicEvolving
+		@Deprecated
+		public <T> SingleOutputStreamOperator<T> with(FlatJoinFunction<T1, T2, T> function) {
+			return (SingleOutputStreamOperator<T>) apply(function);
+		}
+
+		/**
+		 * Completes the join operation with the user function that is executed
+		 * for each combination of elements with the same key in a window.
 		 */
 		public <T> DataStream<T> apply(JoinFunction<T1, T2, T> function, TypeInformation<T> resultType) {
 			//clean the closure
@@ -287,6 +336,22 @@ public class JoinedStreams<T1, T2> {
 					.apply(new JoinCoGroupFunction<>(function), resultType);
 
 		}
+
+		/**
+		 * Completes the join operation with the user function that is executed
+		 * for each combination of elements with the same key in a window.
+		 *
+		 * <p>
+		 *     Note: This is a temporary workaround while the {@link #apply(JoinFunction, TypeInformation)} method has the wrong return type.
+		 * </p>
+		 * @deprecated This method will be replaced by {@link #apply(JoinFunction, TypeInformation)} in Flink 2.0.
+		 * So use the {@link #apply(JoinFunction, TypeInformation)} in the future.
+		 */
+		@PublicEvolving
+		@Deprecated
+		public <T> SingleOutputStreamOperator<T> with(JoinFunction<T1, T2, T> function, TypeInformation<T> resultType) {
+			return (SingleOutputStreamOperator<T>) apply(function, resultType);
+		}
 	}
 	
 	// ------------------------------------------------------------------------