[GitHub] flink pull request #6417: [FLINK-9913][runtime] Improve output serialization...

2018-07-25 Thread zhijiangW
GitHub user zhijiangW opened a pull request: https://github.com/apache/flink/pull/6417 [FLINK-9913][runtime] Improve output serialization only once in RecordWriter ## What is the purpose of the change *This pull request improves the output serialization only once for multi

[GitHub] flink issue #5381: [FLINK-8523][network] Stop assigning floating buffers for...

2018-07-11 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/5381 Yes, I think it can make better use of floating buffers. As last confirmation with Piotr, this PR will not be merged into release-1.5, because there were still some issues not confirmed

[GitHub] flink pull request #6254: [FLINK-9676][network] clarify contracts of BufferL...

2018-07-10 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/6254#discussion_r201293862 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/io/network/partition/consumer/RemoteInputChannel.java --- @@ -594,22 +626,22 @@ public

[GitHub] flink pull request #6254: [FLINK-9676][network] clarify contracts of BufferL...

2018-07-10 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/6254#discussion_r201293052 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/io/network/partition/consumer/RemoteInputChannel.java --- @@ -479,6 +508,9 @@ void

[GitHub] flink issue #6272: [FLINK-9755][network] forward exceptions in RemoteInputCh...

2018-07-10 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/6272 LGTM! ---

[GitHub] flink pull request #6272: [FLINK-9755][network] forward exceptions in Remote...

2018-07-10 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/6272#discussion_r201292263 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/io/network/partition/consumer/RemoteInputChannel.java --- @@ -360,32 +360,45 @@ public

[GitHub] flink issue #6272: [FLINK-9755][network] forward exceptions in RemoteInputCh...

2018-07-10 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/6272 Thanks for fixing this potential bug. It makes sense to handle exception during `notifyBufferAvailable` on listener side. Just some thoughts on my side above. :) ---

[GitHub] flink pull request #6272: [FLINK-9755][network] forward exceptions in Remote...

2018-07-10 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/6272#discussion_r201247182 --- Diff: flink-runtime/src/test/java/org/apache/flink/runtime/io/network/partition/consumer/RemoteInputChannelTest.java --- @@ -687,7 +690,65 @@ public

[GitHub] flink pull request #6272: [FLINK-9755][network] forward exceptions in Remote...

2018-07-10 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/6272#discussion_r201242662 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/io/network/partition/consumer/RemoteInputChannel.java --- @@ -360,32 +360,45 @@ public

[GitHub] flink issue #6271: [FLINK-9766][network][tests] fix cleanup in RemoteInputCh...

2018-07-09 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/6271 LGTM! ---

[GitHub] flink pull request #6257: [FLINK-9676][network] clarify contracts of BufferL...

2018-07-05 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/6257#discussion_r200287796 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/io/network/buffer/LocalBufferPool.java --- @@ -251,27 +257,56 @@ private MemorySegment

[GitHub] flink pull request #6257: [FLINK-9676][network] clarify contracts of BufferL...

2018-07-05 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/6257#discussion_r200287730 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/io/network/buffer/LocalBufferPool.java --- @@ -251,27 +257,56 @@ private MemorySegment

[GitHub] flink issue #6257: [FLINK-9676][network] clarify contracts of BufferListener...

2018-07-05 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/6257 Thanks for fixing this bug, and we also solve this problem in this way. This solution seems more lightweight than the way in [6254](https://github.com/apache/flink/pull/6254), and I also

[GitHub] flink pull request #6257: [FLINK-9676][network] clarify contracts of BufferL...

2018-07-05 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/6257#discussion_r200252253 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/io/network/buffer/LocalBufferPool.java --- @@ -251,27 +257,56 @@ private MemorySegment

[GitHub] flink pull request #6257: [FLINK-9676][network] clarify contracts of BufferL...

2018-07-05 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/6257#discussion_r200251500 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/io/network/buffer/LocalBufferPool.java --- @@ -251,27 +257,56 @@ private MemorySegment

[GitHub] flink pull request #6254: [FLINK-9676][network] clarify contracts of BufferL...

2018-07-04 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/6254#discussion_r200245464 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/io/network/partition/consumer/RemoteInputChannel.java --- @@ -594,22 +626,22 @@ public

[GitHub] flink pull request #6254: [FLINK-9676][network] clarify contracts of BufferL...

2018-07-04 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/6254#discussion_r200247127 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/io/network/partition/consumer/RemoteInputChannel.java --- @@ -479,6 +508,9 @@ void

[GitHub] flink issue #6238: [FLINK-9636][network] fix inconsistency with failed buffe...

2018-07-03 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/6238 👍 ---

[GitHub] flink pull request #6238: [FLINK-9636][network] fix inconsistency with faile...

2018-07-02 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/6238#discussion_r199676237 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/io/network/buffer/NetworkBufferPool.java --- @@ -147,7 +151,12 @@ public void recycle

[GitHub] flink issue #6238: [FLINK-9636][network] fix inconsistency with failed buffe...

2018-07-02 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/6238 Thanks for fixing this bug. :) I think there is another potential bug in polling segments from queue during task canceling process, and I pointed out it in the above codes. ---

[GitHub] flink issue #5923: [FLINK-9253][network] make the maximum floating buffers c...

2018-04-27 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/5923 Thanks for improving it, looks good from my point. :) ---

[GitHub] flink issue #5916: [hotfix][tests] remove redundant rebalance in SuccessAfte...

2018-04-26 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/5916 LGTM :) ---

[GitHub] flink issue #5915: [FLINK-9243][tests] fix flaky SuccessAfterNetworkBuffersF...

2018-04-25 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/5915 The points is already `rebalance()` in line 131, so it is no need to `rebalance()` again in line 137? ---

[GitHub] flink issue #5747: [FLINK-9057][network] fix an NPE when cleaning up before ...

2018-03-22 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/5747 Thanks for fixing this problem. The `notifyReaderCreated` should be called after both views are created correctly, otherwise it will cause inconsistent. LGTM 👍 ---

[GitHub] flink issue #5708: [FLINK-8984][network] Drop taskmanager.exactly-once.block...

2018-03-16 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/5708 Thanks piotr, I agree with it. ---

[GitHub] flink pull request #5317: [FLINK-8458] Add the switch for keeping both the o...

2018-03-01 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/5317#discussion_r171575083 --- Diff: flink-core/src/main/java/org/apache/flink/configuration/TaskManagerOptions.java --- @@ -269,15 +269,21 @@ public static final

[GitHub] flink issue #4529: [FLINK-7428][network] avoid buffer copies when receiving ...

2018-02-27 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/4529 Thanks for telling me the plan of it. I will do that if necessary. :) ---

[GitHub] flink issue #4529: [FLINK-7428][network] avoid buffer copies when receiving ...

2018-02-23 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/4529 Hey @NicoK , would this PR be covered in FLINK-1.5? We experienced the netty direct memory out of memory sometimes in production cased by `extractFrame`, so we expect this improvement. :) ---

[GitHub] flink issue #5317: [FLINK-8458] Add the switch for keeping both the old mode...

2018-02-22 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/5317 @pnowojski , thanks for your messages and I already updated the commit as you suggested. ---

[GitHub] flink issue #5558: [FLINK-8747][bugfix] The tag of waiting for floating buff...

2018-02-22 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/5558 Thanks for your reviews! I already addressed above comments in a separate commit. Regarding the tests I also think the same with you. :) ---

[GitHub] flink pull request #5558: [FLINK-8747][bugfix] The tag of waiting for floati...

2018-02-22 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/5558#discussion_r170148337 --- Diff: flink-runtime/src/test/java/org/apache/flink/runtime/io/network/partition/consumer/RemoteInputChannelTest.java --- @@ -396,32 +399,49 @@ public

[GitHub] flink pull request #5558: [FLINK-8747][bugfix] The tag of waiting for floati...

2018-02-22 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/5558#discussion_r170148132 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/io/network/partition/consumer/RemoteInputChannel.java --- @@ -337,6 +337,11 @@ public int

[GitHub] flink issue #5558: [FLINK-8747][bugfix] The tag of waiting for floating buff...

2018-02-22 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/5558 @NicoK , as I described in the purpose of the change, there are two scenarios for this issue. And I only modified the previous `RemoteInputChannelTest#testAvailableBuffersLessThanRequiredBuffers

[GitHub] flink pull request #5558: [FLINK-8747][bugfix] The tag of waiting for floati...

2018-02-22 Thread zhijiangW
GitHub user zhijiangW opened a pull request: https://github.com/apache/flink/pull/5558 [FLINK-8747][bugfix] The tag of waiting for floating buffers in RemoteInputChannel should be updated properly ## What is the purpose of the change *In credit-based flow control mode

[GitHub] flink issue #5317: [FLINK-8458] Add the switch for keeping both the old mode...

2018-02-20 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/5317 @NicoK , I found most of the codes in this PR are already merged into master by commit `0093bcbe771f296baf3857ef15fe7ec9b22bbc34` in your `FLINK-8425`. Maybe I only need to add the `config.md` in

[GitHub] flink issue #5317: [FLINK-8458] Add the switch for keeping both the old mode...

2018-02-20 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/5317 Yes, I also found that and was rebasing on the latest codes. Because I am on the Spring Festival vacation, it can be ready later today or on Thursday. ---

[GitHub] flink issue #5400: [FLINK-8547][network] Implement CheckpointBarrierHandler ...

2018-02-19 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/5400 @pnowojski , I have changed the `EXACTLY_ONCE_BLOCKING_DATA_ENABLED` as true and squashed the commits. ---

[GitHub] flink issue #5400: [FLINK-8547][network] Implement CheckpointBarrierHandler ...

2018-02-13 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/5400 Thanks for rebasing the conflicts. Yes, the default value can be changed to true after the credit-based is totally merged. If need any changes on my side after all, pls let me know. :) ---

[GitHub] flink issue #5400: [FLINK-8547][network] Implement CheckpointBarrierHandler ...

2018-02-12 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/5400 @pnowojski , I have submitted the updates for above comments. ---

[GitHub] flink pull request #5400: [FLINK-8547][network] Implement CheckpointBarrierH...

2018-02-12 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/5400#discussion_r167582971 --- Diff: flink-streaming-java/src/main/java/org/apache/flink/streaming/runtime/io/BufferBlocker.java --- @@ -0,0 +1,61 @@ +/* + * Licensed to

[GitHub] flink pull request #5400: [FLINK-8547][network] Implement CheckpointBarrierH...

2018-02-12 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/5400#discussion_r167582882 --- Diff: flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/io/BarrierBufferTest.java --- @@ -18,1426 +18,40 @@ package

[GitHub] flink pull request #5400: [FLINK-8547][network] Implement CheckpointBarrierH...

2018-02-12 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/5400#discussion_r167582763 --- Diff: flink-streaming-java/src/main/java/org/apache/flink/streaming/runtime/io/BufferBlocker.java --- @@ -0,0 +1,61 @@ +/* + * Licensed to

[GitHub] flink issue #5400: [FLINK-8547][network] Implement CheckpointBarrierHandler ...

2018-02-09 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/5400 @pnowojski , thanks for suggestions and I totally agree with that. That abstraction indeed makes the code simple. I will update the codes ASAP. ---

[GitHub] flink issue #5400: [FLINK-8547][network] Implement CheckpointBarrierHandler ...

2018-02-07 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/5400 @pnowojski , I have submitted a separate commit to address above comments. ---

[GitHub] flink pull request #5400: [FLINK-8547][network] Implement CheckpointBarrierH...

2018-02-05 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/5400#discussion_r166175837 --- Diff: flink-streaming-java/src/main/java/org/apache/flink/streaming/runtime/io/StreamInputProcessor.java --- @@ -131,10 +131,14 @@ public

[GitHub] flink pull request #5400: [FLINK-8547][network] Implement CheckpointBarrierH...

2018-02-05 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/5400#discussion_r166174743 --- Diff: flink-streaming-java/src/main/java/org/apache/flink/streaming/runtime/io/CreditBasedBarrierBuffer.java --- @@ -0,0 +1,529

[GitHub] flink issue #5400: [FLINK-8547][network] Implement CheckpointBarrierHandler ...

2018-02-05 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/5400 @pnowojski , thanks for reviews! I understand your concerns and I should deduplicate some common utils in these tests. I will do that tomorrow together with other comments! ---

[GitHub] flink pull request #5400: [FLINK-8547][network] Implement CheckpointBarrierH...

2018-02-05 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/5400#discussion_r165998584 --- Diff: flink-streaming-java/src/main/java/org/apache/flink/streaming/runtime/io/StreamInputProcessor.java --- @@ -131,10 +131,14 @@ public

[GitHub] flink pull request #5400: [FLINK-8547][network] Implement CheckpointBarrierH...

2018-02-05 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/5400#discussion_r165997853 --- Diff: flink-streaming-java/src/main/java/org/apache/flink/streaming/runtime/io/CreditBasedBarrierBuffer.java --- @@ -0,0 +1,529

[GitHub] flink pull request #5400: [FLINK-8547][network] Implement CheckpointBarrierH...

2018-02-05 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/5400#discussion_r165983607 --- Diff: flink-streaming-java/src/main/java/org/apache/flink/streaming/runtime/io/CreditBasedBarrierBuffer.java --- @@ -0,0 +1,529

[GitHub] flink pull request #5400: [FLINK-8547][network] Implement CheckpointBarrierH...

2018-02-05 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/5400#discussion_r165983714 --- Diff: flink-streaming-java/src/main/java/org/apache/flink/streaming/runtime/io/CreditBasedBarrierBuffer.java --- @@ -0,0 +1,529

[GitHub] flink pull request #5400: [FLINK-8547][network] Implement CheckpointBarrierH...

2018-02-05 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/5400#discussion_r165983496 --- Diff: flink-core/src/main/java/org/apache/flink/configuration/TaskManagerOptions.java --- @@ -184,6 +184,18 @@ key

[GitHub] flink pull request #5400: [FLINK-8547][network] Implement CheckpointBarrierH...

2018-02-01 Thread zhijiangW
GitHub user zhijiangW opened a pull request: https://github.com/apache/flink/pull/5400 [FLINK-8547][network] Implement CheckpointBarrierHandler not to spill data for exactly-once ## What is the purpose of the change *Currently in exactly-once mode, the BarrierBuffer would

[GitHub] flink pull request #5381: [FLINK-8523][network] Stop assigning floating buff...

2018-01-31 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/5381#discussion_r164993308 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/io/network/partition/consumer/SingleInputGate.java --- @@ -485,6 +494,23 @@ public void

[GitHub] flink issue #5381: [FLINK-8523][network] Stop assigning floating buffers for...

2018-01-31 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/5381 Thanks for reviews and suggestions! :) I will add some unit tests first to verify the related logics. For itcase, I will consider the necessary and feasibility. ---

[GitHub] flink pull request #5381: [FLINK-8523][network] Stop assigning floating buff...

2018-01-31 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/5381#discussion_r164990652 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/io/network/partition/consumer/UnionInputGate.java --- @@ -138,6 +148,24 @@ public void

[GitHub] flink pull request #5381: [FLINK-8523][network] Stop assigning floating buff...

2018-01-31 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/5381#discussion_r164988630 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/io/network/partition/consumer/SingleInputGate.java --- @@ -315,6 +322,7 @@ public void

[GitHub] flink pull request #5381: [FLINK-8523][network] Stop assigning floating buff...

2018-01-31 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/5381#discussion_r164988413 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/io/network/partition/consumer/SingleInputGate.java --- @@ -133,8 +134,13

[GitHub] flink pull request #5381: [FLINK-8523][network] Stop assigning floating buff...

2018-01-31 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/5381#discussion_r164988217 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/io/network/partition/consumer/RemoteInputChannel.java --- @@ -360,8 +360,9 @@ public

[GitHub] flink pull request #5381: [FLINK-8523][network] Stop assigning floating buff...

2018-01-31 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/5381#discussion_r164987333 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/io/network/partition/consumer/InputChannel.java --- @@ -165,6 +168,14 @@ protected void

[GitHub] flink pull request #5381: [FLINK-8523][network] Stop assigning floating buff...

2018-01-31 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/5381#discussion_r164986882 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/io/network/partition/consumer/InputChannel.java --- @@ -66,6 +66,9 @@ /** The

[GitHub] flink issue #5381: [FLINK-8523][network] Stop assigning floating buffers for...

2018-01-30 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/5381 There is one issue I am thinking. After the input channel is unblocked, we can check whether this input channel needs request floating buffer from pool (if `availableNum < numRequiredBuff

[GitHub] flink issue #5381: [FLINK-8523][network] Stop assigning floating buffers for...

2018-01-30 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/5381 @pnowojski I have submitted the whole process in one commit. After you verify the implementation is feasible, I will submit a separate commit for adding unit tests based on this process. ---

[GitHub] flink pull request #5381: [FLINK-8523][network] Stop assigning floating buff...

2018-01-29 Thread zhijiangW
GitHub user zhijiangW opened a pull request: https://github.com/apache/flink/pull/5381 [FLINK-8523][network] Stop assigning floating buffers for blocked input channels in exactly-once mode ## What is the purpose of the change In exactly-once mode, the input channel is set

[GitHub] flink pull request #5317: [FLINK-8458] Add the switch for keeping both the o...

2018-01-22 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/5317#discussion_r163155607 --- Diff: docs/ops/config.md --- @@ -290,6 +290,12 @@ The following parameters configure Flink's JobManager and TaskMan

[GitHub] flink pull request #5317: [FLINK-8458] Add the switch for keeping both the o...

2018-01-22 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/5317#discussion_r163155437 --- Diff: docs/ops/config.md --- @@ -290,6 +290,12 @@ The following parameters configure Flink's JobManager and TaskMan

[GitHub] flink pull request #5317: [FLINK-8458] Add the switch for keeping both the o...

2018-01-19 Thread zhijiangW
GitHub user zhijiangW opened a pull request: https://github.com/apache/flink/pull/5317 [FLINK-8458] Add the switch for keeping both the old mode and the new credit-based mode ## What is the purpose of the change *After the whole feature of credit-based flow control is done

[GitHub] flink pull request #4552: [FLINK-7456][network] Implement Netty sender incom...

2018-01-17 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/4552#discussion_r162266243 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/io/network/partition/SpilledSubpartitionView.java --- @@ -31,7 +31,6 @@ import

[GitHub] flink issue #4552: [FLINK-7456][network] Implement Netty sender incoming pip...

2018-01-17 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/4552 1. Thanks for you FLINK08425. 2. I would have thought the tests for `ResultSubpartition#nextBufferIsEvent` which have already been covered before. The test for `BufferAndBacklog

[GitHub] flink pull request #4552: [FLINK-7456][network] Implement Netty sender incom...

2018-01-17 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/4552#discussion_r162259781 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/io/network/netty/PartitionRequestClientFactory.java --- @@ -164,11 +165,13 @@ private

[GitHub] flink pull request #4552: [FLINK-7456][network] Implement Netty sender incom...

2018-01-17 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/4552#discussion_r162259728 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/io/network/netty/PartitionRequestServerHandler.java --- @@ -82,10 +83,17 @@ protected void

[GitHub] flink pull request #4552: [FLINK-7456][network] Implement Netty sender incom...

2018-01-17 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/4552#discussion_r162259766 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/io/network/netty/PartitionRequestQueue.java --- @@ -125,11 +126,11 @@ private void

[GitHub] flink issue #4552: [FLINK-7456][network] Implement Netty sender incoming pip...

2018-01-16 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/4552 @NicoK , I have submitted the switch for keeping the old mode and the new credit-based mode. ---

[GitHub] flink issue #4552: [FLINK-7456][network] Implement Netty sender incoming pip...

2018-01-15 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/4552 @NicoK , thanks for your reviews! I have submitted all the patches you provided offline to address above issues. 1. Remove `FLINK-8425` from this PR. 2. Do you think I should add

[GitHub] flink pull request #4552: [FLINK-7456][network] Implement Netty sender incom...

2018-01-15 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/4552#discussion_r161667346 --- Diff: flink-tests/src/test/java/org/apache/flink/test/misc/SuccessAfterNetworkBuffersFailureITCase.java --- @@ -59,7 +59,7 @@ public void

[GitHub] flink pull request #4552: [FLINK-7456][network] Implement Netty sender incom...

2018-01-15 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/4552#discussion_r161667234 --- Diff: flink-runtime/src/test/java/org/apache/flink/runtime/taskmanager/TaskCancelAsyncProducerConsumerITCase.java --- @@ -84,7 +84,7 @@ public void

[GitHub] flink issue #4552: [FLINK-7456][network] Implement Netty sender incoming pip...

2018-01-15 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/4552 @NicoK , I have submitted all the modifications based on the patches you provided. The tests for `nextBufferIsEvent` will be added in a new commit tomorrow. ---

[GitHub] flink pull request #4552: [FLINK-7456][network] Implement Netty sender incom...

2018-01-15 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/4552#discussion_r161450876 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/io/network/partition/SpillableSubpartitionView.java --- @@ -199,6 +199,19 @@ public

[GitHub] flink pull request #4552: [FLINK-7456][network] Implement Netty sender incom...

2018-01-10 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/4552#discussion_r160849525 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/io/network/netty/PartitionRequestQueue.java --- @@ -250,10 +304,12 @@ private void

[GitHub] flink pull request #4552: [FLINK-7456][network] Implement Netty sender incom...

2018-01-10 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/4552#discussion_r160849478 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/io/network/netty/PartitionRequestQueue.java --- @@ -88,6 +94,37 @@ public void run

[GitHub] flink pull request #4552: [FLINK-7456][network] Implement Netty sender incom...

2018-01-09 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/4552#discussion_r160371761 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/io/network/netty/PartitionRequestQueue.java --- @@ -88,6 +93,35 @@ public void run

[GitHub] flink pull request #4552: [FLINK-7456][network] Implement Netty sender incom...

2018-01-09 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/4552#discussion_r160333139 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/io/network/netty/PartitionRequestQueue.java --- @@ -88,6 +93,35 @@ public void run

[GitHub] flink pull request #4552: [FLINK-7456][network] Implement Netty sender incom...

2018-01-09 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/4552#discussion_r160333068 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/io/network/netty/PartitionRequestQueue.java --- @@ -88,6 +93,35 @@ public void run

[GitHub] flink issue #4509: [FLINK-7406][network] Implement Netty receiver incoming p...

2018-01-08 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/4509 Already close it ---

[GitHub] flink pull request #4509: [FLINK-7406][network] Implement Netty receiver inc...

2018-01-08 Thread zhijiangW
Github user zhijiangW closed the pull request at: https://github.com/apache/flink/pull/4509 ---

[GitHub] flink pull request #4533: [FLINK-7416][network] Implement Netty receiver out...

2018-01-08 Thread zhijiangW
Github user zhijiangW closed the pull request at: https://github.com/apache/flink/pull/4533 ---

[GitHub] flink issue #4533: [FLINK-7416][network] Implement Netty receiver outgoing p...

2018-01-08 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/4533 Sure, I close it. ---

[GitHub] flink issue #4559: [FLINK-7468][network] Implement sender backlog logic for ...

2017-12-20 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/4559 @NicoK , I have submitted the `hotfix` commit to address above comments. ---

[GitHub] flink pull request #4559: [FLINK-7468][network] Implement sender backlog log...

2017-12-19 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/4559#discussion_r157709904 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/io/network/partition/PipelinedSubpartition.java --- @@ -52,6 +54,10 @@ /** Flag

[GitHub] flink pull request #4559: [FLINK-7468][network] Implement sender backlog log...

2017-12-19 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/4559#discussion_r157706951 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/io/network/partition/ResultSubpartition.java --- @@ -99,6 +82,23 @@ protected Throwable

[GitHub] flink pull request #4559: [FLINK-7468][network] Implement sender backlog log...

2017-12-19 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/4559#discussion_r157694294 --- Diff: flink-runtime/src/test/java/org/apache/flink/runtime/io/network/partition/SubpartitionTestBase.java --- @@ -47,7 +48,14 @@ public void

[GitHub] flink pull request #4559: [FLINK-7468][network] Implement sender backlog log...

2017-12-19 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/4559#discussion_r157693477 --- Diff: flink-runtime/src/test/java/org/apache/flink/runtime/io/network/partition/SpillableSubpartitionTest.java --- @@ -181,10 +182,27 @@ public void

[GitHub] flink pull request #4559: [FLINK-7468][network] Implement sender backlog log...

2017-12-19 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/4559#discussion_r157691096 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/io/network/partition/PipelinedSubpartition.java --- @@ -52,6 +54,10 @@ /** Flag

[GitHub] flink pull request #4559: [FLINK-7468][network] Implement sender backlog log...

2017-12-19 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/4559#discussion_r157686388 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/io/network/partition/ResultSubpartition.java --- @@ -99,6 +82,23 @@ protected Throwable

[GitHub] flink issue #4559: [FLINK-7468][network] Implement sender backlog logic for ...

2017-12-14 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/4559 @NicoK, I have submitted two `[hotfix]` commits for the above issues. One is for `Nullable` annotation and tests of backlog statistics. And the other is for updating backlog in thread-safe

[GitHub] flink issue #4559: [FLINK-7468][network] Implement sender backlog logic for ...

2017-12-12 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/4559 @NicoK , thanks for suggestions. I understand your point of wrapping the buffer and backlog together in a new structure returned by `getNextBuffer()` and it really makes sense for

[GitHub] flink issue #4559: [FLINK-7468][network] Implement sender backlog logic for ...

2017-12-07 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/4559 Or we make the backlog as `AtomicInteger` to keep the current process, otherwise we may need to call `decreaseStatistics` in different parts in the region of `synchronized(buffers)` . What do you

[GitHub] flink issue #4559: [FLINK-7468][network] Implement sender backlog logic for ...

2017-12-07 Thread zhijiangW
Github user zhijiangW commented on the issue: https://github.com/apache/flink/pull/4559 @NicoK @pnowojski , for backlog thread-safe issue, the current implementation is not thread-safe and we should restore my previous implementation that calls `decreaseStatistics` inside the method

[GitHub] flink pull request #4559: [FLINK-7468][network] Implement sender backlog log...

2017-12-07 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/4559#discussion_r155458048 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/io/network/partition/SpillableSubpartitionView.java --- @@ -145,6 +145,10 @@ public Buffer

[GitHub] flink pull request #4559: [FLINK-7468][network] Implement sender backlog log...

2017-12-07 Thread zhijiangW
Github user zhijiangW commented on a diff in the pull request: https://github.com/apache/flink/pull/4559#discussion_r155454935 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/io/network/partition/ResultSubpartitionView.java --- @@ -22,32 +22,57 @@ import

  1   2   3   >