Thanks for the reply.
On Wed, Nov 2, 2016 at 5:19 PM Till Rohrmann wrote:
> Yes you're right. Whenever you have multiple input channels which could
> also be the case if you do a repartitioning between two mappers.
>
> On Tue, Nov 1, 2016 at 11:48 PM, Renjie Liu
> wrote:
>
> Hi, Till:
> I think
Yes you're right. Whenever you have multiple input channels which could
also be the case if you do a repartitioning between two mappers.
On Tue, Nov 1, 2016 at 11:48 PM, Renjie Liu wrote:
> Hi, Till:
> I think the multiple input should include the more general case where
> redistribution happens
Hi, Till:
I think the multiple input should include the more general case where
redistribution happens between subtasks, right? Since in this case we also
need to align check barrier.
Till Rohrmann 于2016年11月1日周二 下午11:05写道:
> The tuples are not buffered until the snapshot is globally complete (a
>
The tuples are not buffered until the snapshot is globally complete (a
snapshot is globally complete iff all operators have successfully taken a
snapshot). They are only buffered until the corresponding checkpoint
barrier on the second input is received. Once this is the case, the
checkpoint barrie
Hi, Till:
By operator with multiple inputs, do you mean inputs from multiple
subtasks?
On Tue, Nov 1, 2016 at 8:56 PM Till Rohrmann wrote:
> Hi Li,
>
> the statement refers to operators with multiple inputs (two in this case).
> With the current implementation you will indeed block one of the in
Sorry the incorrect reply, please ignore this.
On Tue, Nov 1, 2016 at 8:47 PM Renjie Liu wrote:
> Essentially you are right, but the snapshot commit process is
> asynchronous. That's what you have to pay for exactly once semantics.
>
> Li Wang 于2016年11月1日周二 下午3:05写道:
>
> Hi all,
>
> I have a que
Essentially you are right, but the snapshot commit process is asynchronous.
That's what you have to pay for exactly once semantics.
Li Wang 于2016年11月1日周二 下午3:05写道:
> Hi all,
>
> I have a question regarding to the state checkpoint mechanism in Flink. I
> find the statement "Once the last stream h
Hi Till,
Thanks for your prompt reply. I understand that input streams should be aligned
such that a consistent state snapshot can be generated. In my opinion, that
statement indicates that an operator will buffer its output tuples until the
snapshot is committed. I am wondering if my understa
Hi Li,
the statement refers to operators with multiple inputs (two in this case).
With the current implementation you will indeed block one of the inputs
after receiving a checkpoint barrier n until you've received the
corresponding checkpoint barrier n on the other input as well. This is what
we
Hi all,
I have a question regarding to the state checkpoint mechanism in Flink. I find
the statement "Once the last stream has received barrier n, the operator emits
all pending outgoing records, and then emits snapshot n barriers itself” on the
document
https://ci.apache.org/projects/flink/f
10 matches
Mail list logo