Hi Dawid,
thanks for clarification. Having looked into the related classes again, I
agree that exposing the current StreamStatus would severly limit future
developments. I guess to expose it in the Sink, we would need to have a new
WatermarkStatus in eventtime package (as per Eron's suggestion) th
Hey all,
Just a couple of comments from my side as I was called here.
+1 for making stream status just about watermarks.
I observe that AbstractStreamOperator is hardcoded to derive the output
channel's status from the input channel's status. May I suggest
we refactor
"AbstractS
Coming back to my previous comment: I would actually propose to separate
the discussion about whether to expose the WatermarkStatus in the sinks or
not from correcting the StreamStatus and Idleness definition in order to
keep the scope of this FLIP as small as possible. If there is a good reason
to
Hi Martijn,
1. Good question. The watermarks and statuses of the splits are first
aggregated before emitted through the reader. The watermark strategy of the
user is actually applied on all SourceOutputs (=splits). Since one split is
active and one is idle, the watermark of the reader will not adv
Hi all,
I have a couple of questions after studying the FLIP and the docs:
1. What happens when one of the readers has two splits assigned and one of
the splits actually receives data?
2. If I understand it correctly the Kinesis Source uses dynamic shard
discovery by default (so in case of idlen
Hi everyone,
I would be in favour of what Arvid said about not exposing the
WatermarkStatus to the Sink. Unless there is a very strong argument that
this is required I think that keeping this concept internal seems to me the
better choice right now. Moreover, as Arvid said the downstream applicati
Hi Eron,
thank you very much for your feedback.
Please mention that the "temporary status toggle" code will be removed.
>
This code is already removed but there is still some automation of going
idle when temporary no splits are assigned. I will include it in the FLIP.
I agree with adding the ma
This proposal to narrow the definition of idleness to focus on the
event-time clock is great.
Please mention that the "temporary status toggle" code will be removed.
I agree with adding the markActive() functionality, for symmetry. Speaking
of symmetry, could we now include the minor enhancement
Dear devs,
We recently discovered that StreamStatus and Idleness is insufficiently
defined [1], so I drafted a FLIP [3] to amend that situation. It would be
good to hear more opinions on that matter. Ideally, we can make the changes
to 1.14 as some newly added methods are affected.
Best,
Arvid