We are contending with the same issue, as it happens. We have
dozens, and potentially down the line, may need to deal with
thousands of different "time systems" as you put it, and may not
be know at compile time or job start time. In a practical sense,
how could
Thanks for your reply, Xingcan.
On Wed, Nov 8, 2017 at 10:42 PM, Xingcan Cui wrote:
> Hi Shailesh,
>
> actually, the watermarks are generated per partition, but all of them will
> be forcibly aligned to the minimum one during processing. That is decided
> by the semantics of watermark and KeyedS
Hi Shailesh,
actually, the watermarks are generated per partition, but all of them will
be forcibly aligned to the minimum one during processing. That is decided
by the semantics of watermark and KeyedStream, i.e., the watermarks belong
to a whole stream and a stream is made up of different partit
Hi,
I'm working on implementing a use case wherein different physical devices
are sending events, and due to network/power issues, there can be a delay
in receiving events at Flink source. One of the operators within the flink
job is the Pattern operator, and there are certain patterns which are t
s
> keys.
>
> Thanks!
>
>
>
> --
> View this message in context:
> http://apache-flink-user-mailing-list-archive.2336050.n4.nabble.com/Watermarks-per-key-tp11628p11761.html
> Sent from the Apache Flink User Mailing List archive. mailing list archive
> at Nabble.com.
>
timestamps. So is that what's technically preventing assigning
timestamps per key from working?
I'm curious to hear Aljoscha's thoughts on watermark management across keys.
Thanks!
--
View this message in context:
http://apache-flink-user-mailing-list-archive.2336050.n4.nabb
Hi Jordan,
it is not possible to generate watermarks per key. This feature has been
requested a couple of times but I think there are no plans to implement
that.
As far as I understand, the management of watermarks would be quite
expensive (maintaining several watermarks, purging watermarks of
Hi,
I’m designing a streaming job whose elements need to be windowed by event time
across a large set of keys. All elements are read from the same source. Event
time progresses independently across keys. Is it possible to assign timestamps,
and thus generate independent watermarks, per keyed st
gt; .timeWindow(Time.hours(1))
> .allowedLateness(Time.hours(2))
> .apply(new ReadingAggregate(), new AggregateReadings(), new
> AggregateReadings());
>
>
>
> Feedback is very welcome.
>
> best, Stephan
>
>
>
> On 11 Nov 2016, at 00:29
.apply(new ReadingAggregate(), new AggregateReadings(), new
>> AggregateReadings());
>>
>>
>> Feedback is very welcome.
>>
>> best, Stephan
>>
>>
>>
>>
>>> On 11 Nov 2016, at 00:29, Fabian Hueske-2 [via Apache Flin
> best, Stephan
>
>
>
> On 11 Nov 2016, at 00:29, Fabian Hueske-2 [via Apache Flink User Mailing
> List archive.] <[hidden email]
> <http://user/SendEmail.jtp?type=node&node=10179&i=0>> wrote:
>
> Hi Stephan,
>
> I just wrote an answer to your SO q
>> <http://user/SendEmail.jtp?type=node&node=10179&i=0>> wrote:
>>
>
>> Hi Stephan,
>>
>> I just wrote an answer to your SO question.
>>
>> Best, Fabian
>
>>
>> 2016-11-10 11:01 GMT+01:00 Stephan Epping <>
e=10179&i=0>> wrote:
>
> Hi Stephan,
>
> I just wrote an answer to your SO question.
>
> Best, Fabian
>
>
> 2016-11-10 11:01 GMT+01:00 Stephan Epping < href="x-msg://3/user/SendEmail.jtp?type=node&node=10033&i=0"
> target="_top" re
e.] wrote:
>
> Hi Stephan,
>
> I just wrote an answer to your SO question.
>
> Best, Fabian
>
> 2016-11-10 11:01 GMT+01:00 Stephan Epping <[hidden email]
> >:
> Hello,
>
> I found this question in the Nabble archive
> (http://apache-flink-user-mailin
hots? Will they work with the mentioned design?
>>
>> kind regards,
>> Stephan
>
>>> On 11 Nov 2016, at 00:39, Fabian Hueske <>> href="x-msg://10/user/SendEmail.jtp?type=node&node=10094&i=1
>>> "
>>> target="_top
>
> On 11 Nov 2016, at 00:39, Fabian Hueske < href="x-msg://10/user/SendEmail.jtp?type=node&node=10094&i=1"
> target="_top" rel="nofollow" link="external" class="">[hidden email]> wrote:
>
> Hi Stephan,
>
> I
; Stephan
>
>
>> On 11 Nov 2016, at 00:39, Fabian Hueske <[hidden email]
>> > wrote:
>>
>> Hi Stephan,
>>
>> I just wrote an answer to your SO question.
>>
>> Best, Fabian
>>
>> 2016-11-10 11:01 GMT+01:00 Stephan Epping
t;>
>> I found this question in the Nabble archive (
>> http://apache-flink-user-mailing-list-archive.2336050.n4.
>> nabble.com/Maintaining-watermarks-per-key-instead-of-per-
>> operator-instance-tp7288.html) but was unable/dont know how to reply.
>>
>> Here is my que
an
>
> 2016-11-10 11:01 GMT+01:00 Stephan Epping <mailto:stephan.epp...@zweitag.de>>:
> Hello,
>
> I found this question in the Nabble archive
> (http://apache-flink-user-mailing-list-archive.2336050.n4.nabble.com/Maintaining-watermarks-per-key-instead-of-per-operator
Hi Stephan,
I just wrote an answer to your SO question.
Best, Fabian
2016-11-10 11:01 GMT+01:00 Stephan Epping :
> Hello,
>
> I found this question in the Nabble archive (http://apache-flink-user-
> mailing-list-archive.2336050.n4.nabble.com/Maintaining-
> watermarks-per-key
Hello,
I found this question in the Nabble archive
(http://apache-flink-user-mailing-list-archive.2336050.n4.nabble.com/Maintaining-watermarks-per-key-instead-of-per-operator-instance-tp7288.html
<http://apache-flink-user-mailing-list-archive.2336050.n4.nabble.com/Maintaining-watermarks-
Hi,
I think you first have to convert back to a DataStream using .select() or
.flatSelect(). But Till should know more about this, maybe he can help.
Cheers,
Aljoscha
On Thu, 2 Jun 2016 at 19:19 Kanstantsin Kamkou wrote:
> Hi Aljoscha! Is it possible somehow to use the RichXFunction in CEP?
> T
Hi Aljoscha! Is it possible somehow to use the RichXFunction in CEP?
The task is pretty similar, but I have to ignore once the next
triggered event for the same key.
On Wed, Jun 1, 2016 at 2:54 PM, Aljoscha Krettek wrote:
> Hi,
> yeah, in that case per-key watermarks would be useful for you. I w
Hi again Aljoscha,
understood. Thanks for the link. I really like the straightforward approach
concerning storing state. It makes things very easy.
The improvements are very interesting, particularly the composite triggers.
That would significantly improve flexibility.
Kind regards
Leon
1. Ju
Hi,
yeah, in that case per-key watermarks would be useful for you. I won't be
possible to add such a feature, though, due to the (possibly) dynamic
nature of the key space and how watermark tracking works.
You should be able to implement it with relatively low overhead using a
RichFlatMapFunction
Hi Aljoscha,
thanks for the speedy reply.
I am processing measurements delivered by smart meters. I use windows to
gather measurements and calculate values such as average consumption. The key
is simply the meter ID.
The challenge is that meters may undergo network partitioning, under which
t
Hi,
I'm afraid this is impossible with the current design of Flink. Might I ask
what you want to achieve with this? Maybe we can come up with a solution.
-Aljoscha
On Tue, 31 May 2016 at 13:24 wrote:
> My use case primarily concerns applying transformations per key, with the
> keys remaining fi
My use case primarily concerns applying transformations per key, with the
keys remaining fixed throughout the topology. I am using event time for my
windows.
The problem i am currently facing is that watermarks in windows propagate per
operator instance, meaning the operator event time increase
28 matches
Mail list logo