Repository: flink Updated Branches: refs/heads/master d6d0cb97c -> dc0f344f3
[FLINK-9057][network] fix an NPE when cleaning up before requesting a subpartition view In PartitionRequestServerHandler, the view reader was created and immediately afterwards added to the PartitionRequestQueue which would attempt a cleanup of the view reader's subpartition view. This view, however, was currently only created after adding the reader to the PartitionRequestQueue and may thus result in a NullPointerException if the cleanup happens very early in the initialization phase, e.g. due to failures. This closes #5747. Project: http://git-wip-us.apache.org/repos/asf/flink/repo Commit: http://git-wip-us.apache.org/repos/asf/flink/commit/f486a3fd Tree: http://git-wip-us.apache.org/repos/asf/flink/tree/f486a3fd Diff: http://git-wip-us.apache.org/repos/asf/flink/diff/f486a3fd Branch: refs/heads/master Commit: f486a3fd6ed80b67e8eeed9245ad37b6b0be740b Parents: 41ae131 Author: Nico Kruber <n...@data-artisans.com> Authored: Thu Mar 22 13:50:07 2018 +0100 Committer: Till Rohrmann <trohrm...@apache.org> Committed: Wed Mar 28 08:11:03 2018 +0200 ---------------------------------------------------------------------- .../io/network/netty/PartitionRequestServerHandler.java | 6 +++--- 1 file changed, 3 insertions(+), 3 deletions(-) ---------------------------------------------------------------------- http://git-wip-us.apache.org/repos/asf/flink/blob/f486a3fd/flink-runtime/src/main/java/org/apache/flink/runtime/io/network/netty/PartitionRequestServerHandler.java ---------------------------------------------------------------------- diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/io/network/netty/PartitionRequestServerHandler.java b/flink-runtime/src/main/java/org/apache/flink/runtime/io/network/netty/PartitionRequestServerHandler.java index c6a8b1a..e9ee10c 100644 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/io/network/netty/PartitionRequestServerHandler.java +++ b/flink-runtime/src/main/java/org/apache/flink/runtime/io/network/netty/PartitionRequestServerHandler.java @@ -20,9 +20,9 @@ package org.apache.flink.runtime.io.network.netty; import org.apache.flink.runtime.io.network.NetworkSequenceViewReader; import org.apache.flink.runtime.io.network.TaskEventDispatcher; +import org.apache.flink.runtime.io.network.netty.NettyMessage.AddCredit; import org.apache.flink.runtime.io.network.netty.NettyMessage.CancelPartitionRequest; import org.apache.flink.runtime.io.network.netty.NettyMessage.CloseRequest; -import org.apache.flink.runtime.io.network.netty.NettyMessage.AddCredit; import org.apache.flink.runtime.io.network.partition.PartitionNotFoundException; import org.apache.flink.runtime.io.network.partition.ResultPartitionProvider; import org.apache.flink.runtime.io.network.partition.consumer.InputChannelID; @@ -99,12 +99,12 @@ class PartitionRequestServerHandler extends SimpleChannelInboundHandler<NettyMes outboundQueue); } - outboundQueue.notifyReaderCreated(reader); - reader.requestSubpartitionView( partitionProvider, request.partitionId, request.queueIndex); + + outboundQueue.notifyReaderCreated(reader); } catch (PartitionNotFoundException notFound) { respondWithError(ctx, notFound, request.receiverId); }