Re: Snapshot method for custom keyed state checkpointing ?

2021-10-12 Thread Seth Wiesman
Hi Marc, I think you will find this is less efficient than just using keyed state. Remember state backends are local, reading and writing is extremely cheap. HashMapStateBackend is just an in-memory data structure and EmbeddedRocksDBStateBackend only works against local disk. Additionally, the

Re: Snapshot method for custom keyed state checkpointing ?

2021-10-12 Thread Marc LEGER
Hello Nicolaus, Unfortunately, I don't really have the hand on the custom state solution since it is managed by an existing system which cannot be easily modified. What I finally did for the "data state" in my CoFlatMapFunction is to use a* list-style operator state* to store the partitioned

Re: Snapshot method for custom keyed state checkpointing ?

2021-10-12 Thread Nicolaus Weidner
Hi Marc, thanks for clarifying, I had misunderstood some parts. Unfortunately, I don't think there is a way to update keyed state (for multiple keys even) outside of a keyed context. I will ask if someone else has an idea, but allow me to ask one counter-question first: Did you actually run

Re: Snapshot method for custom keyed state checkpointing ?

2021-10-06 Thread Marc LEGER
Hello Nicolaus, Thank you for your quick feedback, sorry if I am not clear enough. Actually in the documented example, the state which is updated in the snapshotState method is an operator state and not a keyed state: *public void initializeState(FunctionInitializationContext context) throws

Re: Snapshot method for custom keyed state checkpointing ?

2021-10-06 Thread Nicolaus Weidner
Hi Marc, I think you can just use keyed state in a CheckpointedFunction. FunctionInitializationContext gives you access to both keyed state and operator state (your stream needs to be keyed, of course). So you could just update your local custom state on regular invocations and update keyed state

Snapshot method for custom keyed state checkpointing ?

2021-10-05 Thread Marc LEGER
Hello, Is there any method available in a RichFunction to be called by Flink with a keyed context each time a checkpoint is triggered please ? It seems that the CheckpointedFunction interface provides such a feature (snapshotState method) but only in case of operator state and it is called in a