Andra Lungu created FLINK-2360:
----------------------------------

             Summary: EOFException
                 Key: FLINK-2360
                 URL: https://issues.apache.org/jira/browse/FLINK-2360
             Project: Flink
          Issue Type: Bug
          Components: Local Runtime
    Affects Versions: 0.10
            Reporter: Andra Lungu
            Priority: Critical


The following code:
https://github.com/andralungu/gelly-partitioning/blob/master/src/main/java/example/NodeSplittingConnectedComponents.java

What the code does, on a very high level:
1). Discovers the skewed nodes in a graph and splits them into subnodes, 
recursively, in levels until we achieve a more uniform degree distribution.
2). Creates a delta iteration that takes the split data set as a solution set. 
On this, it runs the Connected Components Algorithm. At the end of each 
superstep, the partial results computed by the subvertices is gathered back 
into the initial vertex, updating the overall value in the split vertices.
3). Once the iteration converged, the graph is brought back to its initial 
state.

Ran on the twitter follower graph: 
http://twitter.mpi-sws.org/data-icwsm2010.html

With a similar configuration to the one in FLINK-2293. 

Fails with: 
Caused by: java.io.EOFException
        at 
org.apache.flink.runtime.operators.hash.InMemoryPartition$WriteView.nextSegment(InMemoryPartition.java:333)
        at 
org.apache.flink.runtime.memorymanager.AbstractPagedOutputView.advance(AbstractPagedOutputView.java:140)
        at 
org.apache.flink.runtime.memorymanager.AbstractPagedOutputView.writeByte(AbstractPagedOutputView.java:223)
        at 
org.apache.flink.runtime.memorymanager.AbstractPagedOutputView.write(AbstractPagedOutputView.java:173)
        at org.apache.flink.types.StringValue.writeString(StringValue.java:796)
        at 
org.apache.flink.api.common.typeutils.base.StringSerializer.serialize(StringSerializer.java:63)
        at 
org.apache.flink.api.common.typeutils.base.StringSerializer.serialize(StringSerializer.java:28)
        at 
org.apache.flink.api.java.typeutils.runtime.TupleSerializer.serialize(TupleSerializer.java:116)
        at 
org.apache.flink.api.java.typeutils.runtime.TupleSerializer.serialize(TupleSerializer.java:30)
        at 
org.apache.flink.api.java.typeutils.runtime.TupleSerializer.serialize(TupleSerializer.java:116)
        at 
org.apache.flink.api.java.typeutils.runtime.TupleSerializer.serialize(TupleSerializer.java:30)
        at 
org.apache.flink.runtime.operators.hash.InMemoryPartition.appendRecord(InMemoryPartition.java:219)
        at 
org.apache.flink.runtime.operators.hash.CompactingHashTable.insertOrReplaceRecord(CompactingHashTable.java:536)
        at 
org.apache.flink.runtime.operators.hash.CompactingHashTable.buildTableWithUniqueKey(CompactingHashTable.java:347)
        at 
org.apache.flink.runtime.iterative.task.IterationHeadPactTask.readInitialSolutionSet(IterationHeadPactTask.java:209)
        at 
org.apache.flink.runtime.iterative.task.IterationHeadPactTask.run(IterationHeadPactTask.java:270)
        at 
org.apache.flink.runtime.operators.RegularPactTask.invoke(RegularPactTask.java:362)
        at org.apache.flink.runtime.taskmanager.Task.run(Task.java:559)
        at java.lang.Thread.run(Thread.java:722)

Job Manager log:
https://gist.github.com/andralungu/9fc100603ba8d4b8d686



--
This message was sent by Atlassian JIRA
(v6.3.4#6332)

Reply via email to