Details
-
Bug
-
Status: Resolved
-
Major
-
Resolution: Not A Problem
-
0.9, 0.10.0
-
None
Description
The following Exception has been reported by andralungu:
Caused by: java.lang.Exception: The data preparation for task 'CoGroup (CoGroup at groupReduceOnNeighbors(Graph.java:1405))' , caused an error: Error obtaining the sorted input: Thread 'SortMerger spilling thread' terminated due to an exception: Index: 53, Size: 0 at org.apache.flink.runtime.operators.RegularPactTask.run(RegularPactTask.java:476) at org.apache.flink.runtime.operators.RegularPactTask.invoke(RegularPactTask.java:366) at org.apache.flink.runtime.taskmanager.Task.run(Task.java:576) at java.lang.Thread.run(Thread.java:722) Caused by: java.lang.RuntimeException: Error obtaining the sorted input: Thread 'SortMerger spilling thread' terminated due to an exception: Index: 53, Size: 0 at org.apache.flink.runtime.operators.sort.UnilateralSortMerger.getIterator(UnilateralSortMerger.java:607) at org.apache.flink.runtime.operators.RegularPactTask.getInput(RegularPactTask.java:1096) at org.apache.flink.runtime.operators.CoGroupDriver.prepare(CoGroupDriver.java:98) at org.apache.flink.runtime.operators.RegularPactTask.run(RegularPactTask.java:471) ... 3 more Caused by: java.io.IOException: Thread 'SortMerger spilling thread' terminated due to an exception: Index: 53, Size: 0 at org.apache.flink.runtime.operators.sort.UnilateralSortMerger$ThreadBase.run(UnilateralSortMerger.java:784) Caused by: java.lang.IndexOutOfBoundsException: Index: 53, Size: 0 at java.util.ArrayList.rangeCheck(ArrayList.java:604) at java.util.ArrayList.get(ArrayList.java:382) at com.esotericsoftware.kryo.util.MapReferenceResolver.getReadObject(MapReferenceResolver.java:42) at com.esotericsoftware.kryo.Kryo.readReferenceOrNull(Kryo.java:805) at com.esotericsoftware.kryo.Kryo.readClassAndObject(Kryo.java:759) at org.apache.flink.api.java.typeutils.runtime.kryo.KryoSerializer.deserialize(KryoSerializer.java:211) at org.apache.flink.api.java.typeutils.runtime.TupleSerializer.deserialize(TupleSerializer.java:127) at org.apache.flink.api.java.typeutils.runtime.TupleSerializer.deserialize(TupleSerializer.java:30) at org.apache.flink.api.java.typeutils.runtime.TupleSerializer.deserialize(TupleSerializer.java:127) at org.apache.flink.api.java.typeutils.runtime.TupleSerializer.deserialize(TupleSerializer.java:30) at org.apache.flink.runtime.operators.sort.NormalizedKeySorter.writeToOutput(NormalizedKeySorter.java:531) at org.apache.flink.runtime.operators.sort.UnilateralSortMerger$SpillingThread.go(UnilateralSortMerger.java:1328) at org.apache.flink.runtime.operators.sort.UnilateralSortMerger$ThreadBase.run(UnilateralSortMerger.java:781)
See FLINK-2412 for example program and data.