Myasuka commented on a change in pull request #16969:
URL: https://github.com/apache/flink/pull/16969#discussion_r700697901



##########
File path: 
flink-state-backends/flink-statebackend-rocksdb/src/test/java/org/apache/flink/contrib/streaming/state/snapshot/RocksIncrementalSnapshotStrategyTest.java
##########
@@ -0,0 +1,148 @@
+package org.apache.flink.contrib.streaming.state.snapshot;
+
+import org.apache.flink.api.common.state.StateDescriptor;
+import org.apache.flink.api.common.typeutils.base.IntSerializer;
+import org.apache.flink.api.java.tuple.Tuple2;
+import org.apache.flink.contrib.streaming.state.RocksDBKeyedStateBackend;
+import org.apache.flink.contrib.streaming.state.RocksDBOptions;
+import org.apache.flink.contrib.streaming.state.RocksDBStateUploader;
+import org.apache.flink.core.fs.CloseableRegistry;
+import org.apache.flink.runtime.checkpoint.CheckpointOptions;
+import org.apache.flink.runtime.state.ArrayListSerializer;
+import org.apache.flink.runtime.state.CompositeKeySerializationUtils;
+import org.apache.flink.runtime.state.KeyGroupRange;
+import org.apache.flink.runtime.state.RegisteredKeyValueStateBackendMetaInfo;
+import org.apache.flink.runtime.state.StateHandleID;
+import org.apache.flink.runtime.state.TestLocalRecoveryConfig;
+import org.apache.flink.runtime.state.filesystem.FsCheckpointStreamFactory;
+import org.apache.flink.util.ResourceGuard;
+
+import org.junit.Assert;
+import org.junit.Rule;
+import org.junit.Test;
+import org.junit.rules.TemporaryFolder;
+import org.rocksdb.ColumnFamilyDescriptor;
+import org.rocksdb.ColumnFamilyHandle;
+import org.rocksdb.RocksDB;
+import org.rocksdb.RocksDBException;
+
+import java.io.IOException;
+import java.util.ArrayList;
+import java.util.LinkedHashMap;
+import java.util.Set;
+import java.util.SortedMap;
+import java.util.TreeMap;
+import java.util.UUID;
+
+import static org.apache.flink.core.fs.Path.fromLocalFile;
+import static org.apache.flink.core.fs.local.LocalFileSystem.getSharedInstance;
+import static org.powermock.api.mockito.PowerMockito.spy;
+
+/**
+ * Test for verifying the changed logic about notifyCheckpointComplete over
+ * RocksIncrementalSnapshotStrategy is incremental.
+ */
+public class RocksIncrementalSnapshotStrategyTest {
+
+    @Rule public final TemporaryFolder tmp = new TemporaryFolder();
+
+    @Test
+    public void testCheckpointIsIncremental() throws Exception {
+        CloseableRegistry closeableRegistry = new CloseableRegistry();
+
+        // prepare components
+        Tuple2<RocksIncrementalSnapshotStrategy, SortedMap<Long, 
Set<StateHandleID>>> prepareData =
+                prepareCreateSnapshotStrategy(closeableRegistry);
+        RocksIncrementalSnapshotStrategy checkpointSnapshotStrategy = 
prepareData.f0;
+        SortedMap<Long, Set<StateHandleID>> materializedSstFiles = 
prepareData.f1;
+
+        // make checkpoint with id 1
+        RocksIncrementalSnapshotStrategy.IncrementalRocksDBSnapshotResources 
snapshotResources =
+                checkpointSnapshotStrategy.syncPrepareResources(1L);
+
+        checkpointSnapshotStrategy
+                .asyncSnapshot(
+                        snapshotResources,
+                        1L,
+                        1L,
+                        new FsCheckpointStreamFactory(
+                                getSharedInstance(),
+                                fromLocalFile(tmp.newFolder("checkpointsDir")),
+                                fromLocalFile(tmp.newFolder("sharedStateDir")),
+                                100,
+                                100),
+                        CheckpointOptions.forCheckpointWithDefaultLocation())
+                .get(closeableRegistry);
+
+        // notify checkpoint with id 1
+        checkpointSnapshotStrategy.notifyCheckpointComplete(1L);
+
+        // notify savepoint with id 2
+        checkpointSnapshotStrategy.notifyCheckpointComplete(2L);
+
+        // verify stateHandleIDSet is not empty after a savepoint 
notifyComplete
+        Set<StateHandleID> stateHandleIDSet = materializedSstFiles.get(1L);
+        Assert.assertNotNull(stateHandleIDSet);
+        Assert.assertTrue(stateHandleIDSet.size() > 0);
+
+        checkpointSnapshotStrategy.close();
+        closeableRegistry.close();
+    }
+
+    public Tuple2<RocksIncrementalSnapshotStrategy, SortedMap<Long, 
Set<StateHandleID>>>
+            prepareCreateSnapshotStrategy(CloseableRegistry closeableRegistry)
+                    throws IOException, RocksDBException {
+
+        // prepare a rocksDB and put data
+        RocksDB rocksDB = RocksDB.open(tmp.newFolder().getAbsolutePath());

Review comment:
       The created RocksDB instance, column family (instances of `RocksObject`) 
must be closed in the end of the test to avoid native resource leak. Otherwise, 
the test might lead to core dump.

##########
File path: 
flink-state-backends/flink-statebackend-rocksdb/src/test/java/org/apache/flink/contrib/streaming/state/snapshot/RocksIncrementalSnapshotStrategyTest.java
##########
@@ -0,0 +1,148 @@
+package org.apache.flink.contrib.streaming.state.snapshot;
+
+import org.apache.flink.api.common.state.StateDescriptor;
+import org.apache.flink.api.common.typeutils.base.IntSerializer;
+import org.apache.flink.api.java.tuple.Tuple2;
+import org.apache.flink.contrib.streaming.state.RocksDBKeyedStateBackend;
+import org.apache.flink.contrib.streaming.state.RocksDBOptions;
+import org.apache.flink.contrib.streaming.state.RocksDBStateUploader;
+import org.apache.flink.core.fs.CloseableRegistry;
+import org.apache.flink.runtime.checkpoint.CheckpointOptions;
+import org.apache.flink.runtime.state.ArrayListSerializer;
+import org.apache.flink.runtime.state.CompositeKeySerializationUtils;
+import org.apache.flink.runtime.state.KeyGroupRange;
+import org.apache.flink.runtime.state.RegisteredKeyValueStateBackendMetaInfo;
+import org.apache.flink.runtime.state.StateHandleID;
+import org.apache.flink.runtime.state.TestLocalRecoveryConfig;
+import org.apache.flink.runtime.state.filesystem.FsCheckpointStreamFactory;
+import org.apache.flink.util.ResourceGuard;
+
+import org.junit.Assert;
+import org.junit.Rule;
+import org.junit.Test;
+import org.junit.rules.TemporaryFolder;
+import org.rocksdb.ColumnFamilyDescriptor;
+import org.rocksdb.ColumnFamilyHandle;
+import org.rocksdb.RocksDB;
+import org.rocksdb.RocksDBException;
+
+import java.io.IOException;
+import java.util.ArrayList;
+import java.util.LinkedHashMap;
+import java.util.Set;
+import java.util.SortedMap;
+import java.util.TreeMap;
+import java.util.UUID;
+
+import static org.apache.flink.core.fs.Path.fromLocalFile;
+import static org.apache.flink.core.fs.local.LocalFileSystem.getSharedInstance;
+import static org.powermock.api.mockito.PowerMockito.spy;
+
+/**
+ * Test for verifying the changed logic about notifyCheckpointComplete over
+ * RocksIncrementalSnapshotStrategy is incremental.
+ */
+public class RocksIncrementalSnapshotStrategyTest {
+
+    @Rule public final TemporaryFolder tmp = new TemporaryFolder();
+
+    @Test
+    public void testCheckpointIsIncremental() throws Exception {
+        CloseableRegistry closeableRegistry = new CloseableRegistry();
+
+        // prepare components
+        Tuple2<RocksIncrementalSnapshotStrategy, SortedMap<Long, 
Set<StateHandleID>>> prepareData =
+                prepareCreateSnapshotStrategy(closeableRegistry);
+        RocksIncrementalSnapshotStrategy checkpointSnapshotStrategy = 
prepareData.f0;
+        SortedMap<Long, Set<StateHandleID>> materializedSstFiles = 
prepareData.f1;
+
+        // make checkpoint with id 1
+        RocksIncrementalSnapshotStrategy.IncrementalRocksDBSnapshotResources 
snapshotResources =
+                checkpointSnapshotStrategy.syncPrepareResources(1L);
+
+        checkpointSnapshotStrategy
+                .asyncSnapshot(
+                        snapshotResources,
+                        1L,
+                        1L,
+                        new FsCheckpointStreamFactory(
+                                getSharedInstance(),
+                                fromLocalFile(tmp.newFolder("checkpointsDir")),
+                                fromLocalFile(tmp.newFolder("sharedStateDir")),
+                                100,
+                                100),
+                        CheckpointOptions.forCheckpointWithDefaultLocation())
+                .get(closeableRegistry);
+
+        // notify checkpoint with id 1
+        checkpointSnapshotStrategy.notifyCheckpointComplete(1L);
+
+        // notify savepoint with id 2
+        checkpointSnapshotStrategy.notifyCheckpointComplete(2L);
+
+        // verify stateHandleIDSet is not empty after a savepoint 
notifyComplete
+        Set<StateHandleID> stateHandleIDSet = materializedSstFiles.get(1L);
+        Assert.assertNotNull(stateHandleIDSet);
+        Assert.assertTrue(stateHandleIDSet.size() > 0);

Review comment:
       I think this test is bounded to the implementation and you could verify 
the 3rd checkpoint is not incremental. One possible idea is that the returned 
`IncrementalRemoteKeyedStateHandle`s of 1st and 3rd checkpoint do not have 
intersection `sharedState`.




-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org


Reply via email to