zoltar9264 commented on code in PR #22669: URL: https://github.com/apache/flink/pull/22669#discussion_r1220936492
########## flink-state-backends/flink-statebackend-rocksdb/src/main/java/org/apache/flink/contrib/streaming/state/snapshot/RocksDBSnapshotStrategyBase.java: ########## @@ -395,18 +394,16 @@ public void release() { /** Previous snapshot with uploaded sst files. */ protected static class PreviousSnapshot { - @Nullable private final Map<StateHandleID, Long> confirmedSstFiles; + @Nullable private final Map<StateHandleID, StreamStateHandle> confirmedSstFiles; - protected PreviousSnapshot(@Nullable Map<StateHandleID, Long> confirmedSstFiles) { + protected PreviousSnapshot( + @Nullable Map<StateHandleID, StreamStateHandle> confirmedSstFiles) { this.confirmedSstFiles = confirmedSstFiles; } protected Optional<StreamStateHandle> getUploaded(StateHandleID stateHandleID) { if (confirmedSstFiles != null && confirmedSstFiles.containsKey(stateHandleID)) { - // we introduce a placeholder state handle, that is replaced with the - // original from the shared state registry (created from a previous checkpoint) - return Optional.of( - new PlaceholderStreamStateHandle(confirmedSstFiles.get(stateHandleID))); Review Comment: Thanks for your patience in explaining @rkhachatryan , I totally agree with your last proposal. And I realized in coding that maybe we shouldn't store SharedStateRegistryKey in IncrementalRemoteKeyedStateHandle. Because SharedStateRegistryKey is only used during registration and can be calculated from physical id. More importantly, when the remote state file is relocated (savepoint is relocatable), the physical id will change accordingly. The [aforementioned change of IncrementalRemoteKeyedStateHandle](https://github.com/apache/flink/pull/22669#discussion_r1213882810) should look like this: ``` /** Shared state in the incremental checkpoint. */ private final List<HandleAndLocalPath> sharedState; /** Private state in the incremental checkpoint. */ private final List<HandleAndLocalPath> privateState; private static final class HandleAndLocalPath { StreamStateHandle handle; String localPath; } ``` This way, we don't need to introduce a new serialized version for IncrementalRemoteKeyedStateHandle in `MetadataV2V3SerializerBase` either. Because `HandleAndLocalPath` and `Map.Entry<StateHandleID, StreamStateHandle>` can use exactly the same serialization form. WDYT ? -- 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