Hi!
It is not documented. If you're interested you can refer to OperatorChain
class and StreamTask class.
Hi Caizhi,
Thanks for the reply. Is this documented somewhere?
On Tue, Aug 3, 2021 at 10:56 AM Caizhi Weng wrote:
> Hi!
>
> As far as I know, this output.collect thingy is not thread safe, and you
> should never run your operator's main logic (from reading in the record to
> writing the results
Hi!
As far as I know, this output.collect thingy is not thread safe, and you
should never run your operator's main logic (from reading in the record to
writing the results out) in a separated thread. Flink's runtime expect the
whole operator chain to run in a single thread.
Yuval Itzchakov 于2021
Hi,
Flink 1.13.1
Scala 2.12.4
I have an implementation of an AbstractStreamOperator, where in it's
processElement function enqueues an element to a queue which is pooled from
a background thread.
When processing the elements in the background, I use the Output class to
emit elements downstream w
2. Is there any scenario that the developers need to worry about
> process/thread safety when using state?
Few things here:
* do not share mutable static variables without synchronization.
* try to minimize/avoid doing long blocking calls. Use asynchronous
API if applicable.
3. can I
dicates that the persistedValue can be process safe (cross multiple
workers) and thread safe (inside
a worker, e.g. timer callback).
Could you please add some clarification on the questions below?
1. What's the design (briefly) for persisted state process/thread safety?
2. Is there any scenario that t
As others have mentioned already, it is true that method calls on operators
(e.g. processing events and snapshotting state) will not concurrently
happen.
As for your findings in reading through the documentation, that might be a
hint that we could add a bit more explanation mentioning this.
Could
As the java doc suggests it seems operator method and snapshot checkpoint
are accessed by two different threads
https://github.com/apache/flink/blob/release-1.10.0/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/functions/source/SourceFunction.java#L39-L62
On Thu, May 7, 2020 at
I’ve seen a few mailing list posts (including this one) that say Flink
guarantees there is no concurrent access to operator methods (e.g. flatMap,
snapshotState, etc.) and thus synchronization isn’t needed when writing
operators that support checkpointing. I was trying to find a place in the
of
Hi Aaron,
It is thread safe since the state snapshot happens in the same thread with
the user function.
Thanks,
Zhu Zhu
Aaron Langford 于2019年12月19日周四 上午11:25写道:
> Hello Flink Community,
>
> I'm hoping to verify some understanding:
>
> If I have a function with managed state, I'm wondering if a
Hello Flink Community,
I'm hoping to verify some understanding:
If I have a function with managed state, I'm wondering if a checkpoint will
ever be taken while a function is mutating state. I'll try to illustrate
the situation I'm hoping to be safe from:
Happy Path:
t0 -> processFunction invoked
e
https://ci.apache.org/projects/flink/flink-docs-release-1.3/dev/stream/process_function.html use
ValueState and does not use synchronization. Is it because
processElement and onTimer are executed in the same thread and so are
thread safe ?
Also, I could not find any thread safety documentation
jects/flink/flink-docs-release-1.3/dev/stream/process_function.html
use ValueState and does not use synchronization. Is it because processElement
and onTimer are executed in the same thread and so are thread safe ?
Also, I could not find any thread safety documentation about MapState.
Thanks.
Mans
gt;>
>> The example
>> https://ci.apache.org/projects/flink/flink-docs-release-1.3/dev/stream/process_function.html
>> use ValueState and does not use synchronization. Is it because
>> processElement and onTimer are executed in the same thread and so are thread
>> safe ?
>>
>> Also, I could not find any thread safety documentation about MapState.
>>
>> Thanks.
>>
>> Mans
>>
>>
hread safe ?
Also, I could not find any thread safety documentation about MapState.
Thanks.
Mans
/projects/flink/flink-docs-release-1.3/dev/stream/process_function.html
use ValueState and does not use synchronization. Is it because processElement
and onTimer are executed in the same thread and so are thread safe ?
Also, I could not find any thread safety documentation about MapState.
Thanks
multiple functions,
when they are chained. Have a look at the "Concepts" section in the Flink
docs.
On Mon, Feb 27, 2017 at 6:19 PM, Mohit Anchlia
wrote:
> Trying to understand what parts of flink have thread safety built in them.
> Key question is, are the objects created in flin
Trying to understand what parts of flink have thread safety built in them.
Key question is, are the objects created in flink shared between threads
(slots)? For eg: if I create a sink function and open a file is that shared
between threads?
18 matches
Mail list logo