[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168229878 --- Diff: flink-runtime/src/test/java/org/apache/flink/runtime/state/StateBackendTestBase.java --- @@ -3404,12 +3405,16 @@ public String fold(String

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread sihuazhou
Github user sihuazhou commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168193980 --- Diff: flink-contrib/flink-statebackend-rocksdb/src/main/java/org/apache/flink/contrib/streaming/state/RocksDBKeyedStateBackend.java --- @@ -336,1697 +

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread sihuazhou
Github user sihuazhou commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168198204 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/state/TaskLocalStateStoreImpl.java --- @@ -0,0 +1,298 @@ +/* + * Licensed to the Ap

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread sihuazhou
Github user sihuazhou commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168193098 --- Diff: flink-contrib/flink-statebackend-rocksdb/src/main/java/org/apache/flink/contrib/streaming/state/RocksDBKeyedStateBackend.java --- @@ -336,1697 +

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread sihuazhou
Github user sihuazhou commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168191066 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/state/TaskLocalStateStoreImpl.java --- @@ -0,0 +1,298 @@ +/* + * Licensed to the Ap

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread sihuazhou
Github user sihuazhou commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168189852 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/state/TaskLocalStateStoreImpl.java --- @@ -0,0 +1,298 @@ +/* + * Licensed to the Ap

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread sihuazhou
Github user sihuazhou commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168184247 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/state/TaskLocalStateStoreImpl.java --- @@ -0,0 +1,298 @@ +/* + * Licensed to the Ap

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168185050 --- Diff: flink-contrib/flink-statebackend-rocksdb/src/main/java/org/apache/flink/contrib/streaming/state/RocksDBStateBackend.java --- @@ -686,4 +714

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168184599 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/state/heap/HeapKeyedStateBackend.java --- @@ -630,19 +506,210 @@ public int numStateEn

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168184825 --- Diff: flink-contrib/flink-statebackend-rocksdb/src/main/java/org/apache/flink/contrib/streaming/state/RocksDBKeyedStateBackend.java --- @@ -363,1

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168180636 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/state/SnapshotStrategy.java --- @@ -0,0 +1,50 @@ +/* + * Licensed to the Apach

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168180080 --- Diff: flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/tasks/LocalStateForwardingTest.java --- @@ -156,30 +157,38 @@ public

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168179062 --- Diff: flink-runtime/src/test/java/org/apache/flink/runtime/state/TaskStateManagerImplTest.java --- @@ -97,6 +104,39 @@ public void testStateRepor

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168179034 --- Diff: flink-runtime/src/test/java/org/apache/flink/runtime/state/TaskStateManagerImplTest.java --- @@ -97,6 +104,39 @@ public void testStateRepor

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168177861 --- Diff: flink-runtime/src/test/java/org/apache/flink/runtime/state/TaskStateManagerImplTest.java --- @@ -25,17 +25,24 @@ import org.apache.fli

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168177813 --- Diff: flink-runtime/src/test/java/org/apache/flink/runtime/state/TaskExecutorLocalStateStoresManagerTest.java --- @@ -0,0 +1,122 @@ +/* +

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168176949 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/taskexecutor/TaskManagerServices.java --- @@ -197,7 +201,13 @@ public static TaskManag

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168176995 --- Diff: flink-runtime/src/test/java/org/apache/flink/runtime/state/TaskExecutorLocalStateStoresManagerTest.java --- @@ -0,0 +1,122 @@ +/* +

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168176306 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/state/filesystem/FsStateBackend.java --- @@ -501,4 +529,53 @@ public String toString()

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168176031 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/state/TaskStateManager.java --- @@ -60,4 +62,9 @@ void reportTaskStateSnapshots(

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168169824 --- Diff: flink-contrib/flink-statebackend-rocksdb/src/main/java/org/apache/flink/contrib/streaming/state/RocksDBKeyedStateBackend.java --- @@ -1016,

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168169675 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/state/DefaultOperatorStateBackend.java --- @@ -325,8 +325,7 @@ public OperatorStateHan

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168169548 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/state/SnapshotResult.java --- @@ -0,0 +1,83 @@ +package org.apache.flink.runtime.s

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168169484 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/state/StateSnapshotContextSynchronousImpl.java --- @@ -130,7 +136,7 @@ public Operator

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168169451 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/state/StateSnapshotContextSynchronousImpl.java --- @@ -109,14 +109,20 @@ public Operat

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168169295 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/state/TaskStateManager.java --- @@ -47,10 +47,11 @@ * @param checkpointMetric

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168169149 --- Diff: flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/tasks/StreamTaskTerminationTest.java --- @@ -227,21 +229,13 @@ protec

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168169168 --- Diff: flink-streaming-java/src/main/java/org/apache/flink/streaming/runtime/tasks/StreamTask.java --- @@ -916,57 +916,78 @@ private void handleEx

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168169207 --- Diff: flink-streaming-java/src/main/java/org/apache/flink/streaming/runtime/tasks/StreamTask.java --- @@ -859,56 +861,77 @@ public void run() {

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168169266 --- Diff: flink-runtime/src/test/java/org/apache/flink/runtime/state/StateBackendTestBase.java --- @@ -3404,12 +3405,16 @@ public String fold(String

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168169181 --- Diff: flink-streaming-java/src/test/java/org/apache/flink/streaming/api/operators/OperatorSnapshotFuturesTest.java --- @@ -46,20 +47,28 @@ public

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168169074 --- Diff: flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/tasks/StreamTaskTerminationTest.java --- @@ -227,21 +229,13 @@ protec

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168168982 --- Diff: flink-runtime/src/test/java/org/apache/flink/runtime/state/TestTaskStateManager.java --- @@ -67,8 +69,11 @@ public TestTaskStateManager(

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168168859 --- Diff: flink-core/src/test/java/org/apache/flink/util/MethodForwardingTestUtil.java --- @@ -0,0 +1,146 @@ +/* + * Licensed to the Apache So

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168168842 --- Diff: flink-core/src/test/java/org/apache/flink/util/MethodForwardingTestUtil.java --- @@ -0,0 +1,146 @@ +/* + * Licensed to the Apache So

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168168810 --- Diff: flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/StateObjectCollectionTest.java --- @@ -0,0 +1,64 @@ +/* + * License

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168168726 --- Diff: flink-contrib/flink-statebackend-rocksdb/src/main/java/org/apache/flink/contrib/streaming/state/RocksDBStateBackend.java --- @@ -104,10 +10

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168168796 --- Diff: flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/tasks/LocalStateForwardingTest.java --- @@ -0,0 +1,213 @@ +/*

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168168770 --- Diff: flink-core/src/test/java/org/apache/flink/util/MethodForwardingTestUtil.java --- @@ -0,0 +1,146 @@ +/* + * Licensed to the Apache So

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168168705 --- Diff: flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/tasks/LocalStateForwardingTest.java --- @@ -0,0 +1,213 @@ +/*

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168168634 --- Diff: flink-contrib/flink-statebackend-rocksdb/src/main/java/org/apache/flink/contrib/streaming/state/RocksDBStateBackend.java --- @@ -104,10 +10

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168168472 --- Diff: flink-contrib/flink-statebackend-rocksdb/src/main/java/org/apache/flink/contrib/streaming/state/RocksDBStateBackend.java --- @@ -686,4 +714

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168168418 --- Diff: flink-core/src/main/java/org/apache/flink/configuration/ConfigConstants.java --- @@ -316,6 +316,11 @@ */ public static fina

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-14 Thread StefanRRichter
Github user StefanRRichter commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r168168381 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/state/TaskLocalStateStore.java --- @@ -46,26 +52,63 @@ /** */ private

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-09 Thread bowenli86
Github user bowenli86 commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r167356160 --- Diff: docs/ops/state/large_state_tuning.md --- @@ -234,4 +234,97 @@ Compression can be activated through the `ExecutionConfig`: **Notice:** The co

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-09 Thread bowenli86
Github user bowenli86 commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r167355160 --- Diff: docs/ops/state/large_state_tuning.md --- @@ -234,4 +234,97 @@ Compression can be activated through the `ExecutionConfig`: **Notice:** The co

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-09 Thread bowenli86
Github user bowenli86 commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r167389639 --- Diff: docs/ops/state/large_state_tuning.md --- @@ -234,4 +234,97 @@ Compression can be activated through the `ExecutionConfig`: **Notice:** The co

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-09 Thread bowenli86
Github user bowenli86 commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r167357375 --- Diff: docs/ops/state/large_state_tuning.md --- @@ -234,4 +234,97 @@ Compression can be activated through the `ExecutionConfig`: **Notice:** The co

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-09 Thread bowenli86
Github user bowenli86 commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r167356732 --- Diff: docs/ops/state/large_state_tuning.md --- @@ -234,4 +234,97 @@ Compression can be activated through the `ExecutionConfig`: **Notice:** The co

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-09 Thread bowenli86
Github user bowenli86 commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r167354757 --- Diff: docs/ops/state/large_state_tuning.md --- @@ -234,4 +234,97 @@ Compression can be activated through the `ExecutionConfig`: **Notice:** The co

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-09 Thread bowenli86
Github user bowenli86 commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r167357486 --- Diff: docs/ops/state/large_state_tuning.md --- @@ -234,4 +234,97 @@ Compression can be activated through the `ExecutionConfig`: **Notice:** The co

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-09 Thread bowenli86
Github user bowenli86 commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r167356422 --- Diff: docs/ops/state/large_state_tuning.md --- @@ -234,4 +234,97 @@ Compression can be activated through the `ExecutionConfig`: **Notice:** The co

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-09 Thread bowenli86
Github user bowenli86 commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r167389638 --- Diff: docs/ops/state/large_state_tuning.md --- @@ -234,4 +234,97 @@ Compression can be activated through the `ExecutionConfig`: **Notice:** The co

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-09 Thread bowenli86
Github user bowenli86 commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r167354956 --- Diff: docs/ops/state/large_state_tuning.md --- @@ -234,4 +234,97 @@ Compression can be activated through the `ExecutionConfig`: **Notice:** The co

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-09 Thread bowenli86
Github user bowenli86 commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r167355583 --- Diff: docs/ops/state/large_state_tuning.md --- @@ -234,4 +234,97 @@ Compression can be activated through the `ExecutionConfig`: **Notice:** The co

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-09 Thread bowenli86
Github user bowenli86 commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r167356561 --- Diff: docs/ops/state/large_state_tuning.md --- @@ -234,4 +234,97 @@ Compression can be activated through the `ExecutionConfig`: **Notice:** The co

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-09 Thread bowenli86
Github user bowenli86 commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r167357164 --- Diff: docs/ops/state/large_state_tuning.md --- @@ -234,4 +234,97 @@ Compression can be activated through the `ExecutionConfig`: **Notice:** The co

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-09 Thread bowenli86
Github user bowenli86 commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r167357036 --- Diff: docs/ops/state/large_state_tuning.md --- @@ -234,4 +234,97 @@ Compression can be activated through the `ExecutionConfig`: **Notice:** The co

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165349688 --- Diff: flink-contrib/flink-statebackend-rocksdb/src/main/java/org/apache/flink/contrib/streaming/state/RocksDBKeyedStateBackend.java --- @@ -363,169

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165349012 --- Diff: flink-contrib/flink-statebackend-rocksdb/src/main/java/org/apache/flink/contrib/streaming/state/RocksDBKeyedStateBackend.java --- @@ -363,169

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165344648 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/state/heap/HeapKeyedStateBackend.java --- @@ -630,19 +506,210 @@ public int numStateEntr

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165343607 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/state/SnapshotStrategy.java --- @@ -0,0 +1,50 @@ +/* + * Licensed to the Apache

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165343021 --- Diff: flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/tasks/LocalStateForwardingTest.java --- @@ -156,30 +157,38 @@ public vo

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165342796 --- Diff: flink-runtime/src/test/java/org/apache/flink/runtime/util/JvmExitOnFatalErrorTest.java --- @@ -137,9 +138,12 @@ public static void main(Strin

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165342545 --- Diff: flink-runtime/src/test/java/org/apache/flink/runtime/state/TaskStateManagerImplTest.java --- @@ -97,6 +104,39 @@ public void testStateReporti

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165342341 --- Diff: flink-runtime/src/test/java/org/apache/flink/runtime/state/TaskStateManagerImplTest.java --- @@ -97,6 +104,39 @@ public void testStateReporti

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165342139 --- Diff: flink-runtime/src/test/java/org/apache/flink/runtime/state/TaskStateManagerImplTest.java --- @@ -25,17 +25,24 @@ import org.apache.flink

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165341901 --- Diff: flink-runtime/src/test/java/org/apache/flink/runtime/state/TaskExecutorLocalStateStoresManagerTest.java --- @@ -0,0 +1,122 @@ +/* + *

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165341653 --- Diff: flink-runtime/src/test/java/org/apache/flink/runtime/state/TaskExecutorLocalStateStoresManagerTest.java --- @@ -0,0 +1,122 @@ +/* + *

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165341397 --- Diff: flink-runtime/src/test/java/org/apache/flink/runtime/state/TaskExecutorLocalStateStoresManagerTest.java --- @@ -0,0 +1,122 @@ +/* + *

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165341071 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/taskexecutor/TaskManagerServices.java --- @@ -197,7 +201,13 @@ public static TaskManager

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165340438 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/state/filesystem/FsStateBackend.java --- @@ -501,4 +529,53 @@ public String toString() {

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165340246 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/state/TaskStateManager.java --- @@ -60,4 +62,9 @@ void reportTaskStateSnapshots(

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165340082 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/state/TaskLocalStateStore.java --- @@ -46,26 +52,63 @@ /** */ private fi

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165332129 --- Diff: flink-core/src/main/java/org/apache/flink/configuration/ConfigConstants.java --- @@ -316,6 +316,11 @@ */ public static final

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165331598 --- Diff: flink-contrib/flink-statebackend-rocksdb/src/main/java/org/apache/flink/contrib/streaming/state/RocksDBStateBackend.java --- @@ -686,4 +714,5

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165331645 --- Diff: flink-contrib/flink-statebackend-rocksdb/src/main/java/org/apache/flink/contrib/streaming/state/RocksDBStateBackend.java --- @@ -686,4 +714,5

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165329478 --- Diff: flink-contrib/flink-statebackend-rocksdb/src/main/java/org/apache/flink/contrib/streaming/state/RocksDBStateBackend.java --- @@ -104,10 +107,

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165329369 --- Diff: flink-contrib/flink-statebackend-rocksdb/src/main/java/org/apache/flink/contrib/streaming/state/RocksDBStateBackend.java --- @@ -104,10 +107,

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165326694 --- Diff: flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/tasks/LocalStateForwardingTest.java --- @@ -0,0 +1,213 @@ +/* +

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165325793 --- Diff: flink-core/src/test/java/org/apache/flink/util/MethodForwardingTestUtil.java --- @@ -0,0 +1,146 @@ +/* + * Licensed to the Apache Soft

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165323982 --- Diff: flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/tasks/LocalStateForwardingTest.java --- @@ -0,0 +1,213 @@ +/* +

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165323823 --- Diff: flink-runtime/src/test/java/org/apache/flink/runtime/checkpoint/StateObjectCollectionTest.java --- @@ -0,0 +1,64 @@ +/* + * Licensed

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165323575 --- Diff: flink-core/src/test/java/org/apache/flink/util/MethodForwardingTestUtil.java --- @@ -0,0 +1,146 @@ +/* + * Licensed to the Apache Soft

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165322528 --- Diff: flink-core/src/test/java/org/apache/flink/util/MethodForwardingTestUtil.java --- @@ -0,0 +1,146 @@ +/* + * Licensed to the Apache Soft

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165321013 --- Diff: flink-runtime/src/test/java/org/apache/flink/runtime/state/TestTaskStateManager.java --- @@ -67,8 +69,11 @@ public TestTaskStateManager(

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165318377 --- Diff: flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/tasks/StreamTaskTerminationTest.java --- @@ -227,21 +229,13 @@ protecte

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165318246 --- Diff: flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/tasks/StreamTaskTerminationTest.java --- @@ -227,21 +229,13 @@ protecte

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165317571 --- Diff: flink-streaming-java/src/main/java/org/apache/flink/streaming/runtime/tasks/StreamTask.java --- @@ -916,57 +916,78 @@ private void handleExec

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165315816 --- Diff: flink-streaming-java/src/test/java/org/apache/flink/streaming/api/operators/OperatorSnapshotFuturesTest.java --- @@ -46,20 +47,28 @@ public v

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165315625 --- Diff: flink-streaming-java/src/main/java/org/apache/flink/streaming/runtime/tasks/StreamTask.java --- @@ -859,56 +861,77 @@ public void run() {

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165309957 --- Diff: flink-streaming-java/src/main/java/org/apache/flink/streaming/api/operators/OperatorSnapshotFinalizer.java --- @@ -0,0 +1,83 @@ +/* +

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165309670 --- Diff: flink-streaming-java/src/main/java/org/apache/flink/streaming/api/operators/OperatorSnapshotFinalizer.java --- @@ -0,0 +1,83 @@ +/* +

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165309220 --- Diff: flink-runtime/src/test/java/org/apache/flink/runtime/state/StateBackendTestBase.java --- @@ -3404,12 +3405,16 @@ public String fold(String ac

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165308709 --- Diff: flink-runtime/src/test/java/org/apache/flink/runtime/state/StateBackendTestBase.java --- @@ -3404,12 +3405,16 @@ public String fold(String ac

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165308534 --- Diff: flink-runtime/src/test/java/org/apache/flink/runtime/state/StateBackendTestBase.java --- @@ -3404,12 +3405,16 @@ public String fold(String ac

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165307761 --- Diff: flink-runtime/src/test/java/org/apache/flink/runtime/state/OperatorStateBackendTest.java --- @@ -355,10 +355,11 @@ public void testSnapshotEm

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165301094 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/state/TaskStateManager.java --- @@ -47,10 +47,11 @@ * @param checkpointMetrics

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-02-01 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165282440 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/state/DefaultOperatorStateBackend.java --- @@ -325,8 +325,7 @@ public OperatorStateHandl

[GitHub] flink pull request #5239: [FLINK-8360] Implement task-local state recovery

2018-01-31 Thread tillrohrmann
Github user tillrohrmann commented on a diff in the pull request: https://github.com/apache/flink/pull/5239#discussion_r165275719 --- Diff: flink-runtime/src/main/java/org/apache/flink/runtime/state/StateSnapshotContextSynchronousImpl.java --- @@ -109,14 +109,20 @@ public Operator

<    1   2   3   4   >