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 2015/04/09 22:13:32 UTC

[2/3] flink git commit: [FLINK-1845] [runtime] Fixup ReusingKeyGroupedIterator value iterators in NonReusingSortMergeCoGroupIterator.

[FLINK-1845] [runtime] Fixup ReusingKeyGroupedIterator value iterators in NonReusingSortMergeCoGroupIterator.

This closes #559


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

Branch: refs/heads/master
Commit: 8740d1e77b82365c009c0fff37fd24c24a67f8bc
Parents: 1cff478
Author: Alexander Alexandrov <al...@gmail.com>
Authored: Wed Apr 1 18:08:26 2015 +0200
Committer: Stephan Ewen <se...@apache.org>
Committed: Thu Apr 9 21:00:38 2015 +0200

----------------------------------------------------------------------
 .../sort/NonReusingSortMergeCoGroupIterator.java          | 10 +++++-----
 1 file changed, 5 insertions(+), 5 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/flink/blob/8740d1e7/flink-runtime/src/main/java/org/apache/flink/runtime/operators/sort/NonReusingSortMergeCoGroupIterator.java
----------------------------------------------------------------------
diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/operators/sort/NonReusingSortMergeCoGroupIterator.java b/flink-runtime/src/main/java/org/apache/flink/runtime/operators/sort/NonReusingSortMergeCoGroupIterator.java
index 3d6b744..699d243 100644
--- a/flink-runtime/src/main/java/org/apache/flink/runtime/operators/sort/NonReusingSortMergeCoGroupIterator.java
+++ b/flink-runtime/src/main/java/org/apache/flink/runtime/operators/sort/NonReusingSortMergeCoGroupIterator.java
@@ -22,7 +22,7 @@ import org.apache.flink.api.common.typeutils.TypeComparator;
 import org.apache.flink.api.common.typeutils.TypePairComparator;
 import org.apache.flink.api.common.typeutils.TypeSerializer;
 import org.apache.flink.runtime.operators.util.CoGroupTaskIterator;
-import org.apache.flink.runtime.util.ReusingKeyGroupedIterator;
+import org.apache.flink.runtime.util.NonReusingKeyGroupedIterator;
 import org.apache.flink.util.MutableObjectIterator;
 
 import java.io.IOException;
@@ -45,9 +45,9 @@ public class NonReusingSortMergeCoGroupIterator<T1, T2> implements CoGroupTaskIt
 
 	private TypePairComparator<T1, T2> comp;
 
-	private ReusingKeyGroupedIterator<T1> iterator1;
+	private NonReusingKeyGroupedIterator<T1> iterator1;
 
-	private ReusingKeyGroupedIterator<T2> iterator2;
+	private NonReusingKeyGroupedIterator<T2> iterator2;
 
 	// --------------------------------------------------------------------------------------------
 
@@ -61,8 +61,8 @@ public class NonReusingSortMergeCoGroupIterator<T1, T2> implements CoGroupTaskIt
 
 		this.comp = pairComparator;
 
-		this.iterator1 = new ReusingKeyGroupedIterator<T1>(input1, serializer1, groupingComparator1);
-		this.iterator2 = new ReusingKeyGroupedIterator<T2>(input2, serializer2, groupingComparator2);
+		this.iterator1 = new NonReusingKeyGroupedIterator<T1>(input1, groupingComparator1);
+		this.iterator2 = new NonReusingKeyGroupedIterator<T2>(input2, groupingComparator2);
 	}
 
 	@Override