iour-configuration
>
> Best
> Yun Tang
> ------
> *From:* Ahmad Hassan
> *Sent:* Tuesday, January 28, 2020 17:43
> *To:* user
> *Subject:* Re: Flink RocksDB logs filling up disk space
>
> Hi Yun,
>
> Thank you for pointing that out. In our production landscapes with
al of
> checkpoint, even you could avoid to record too many logs, and I don't think
> current checkpoint configuration is appropriate.
>
> Best
> Yun Tang
> ------
> *From:* Ahmad Hassan
> *Sent:* Monday, January 27, 2020 20:22
> *To:* user
&
Thanks Chesnay!
On Mon, 27 Jan 2020 at 11:29, Chesnay Schepler wrote:
> Please see https://issues.apache.org/jira/browse/FLINK-15068
>
> On 27/01/2020 12:22, Ahmad Hassan wrote:
>
> Hello,
>
> In our production systems, we see that flink rocksdb checkpoint IO logs
> a
Hello,
In our production systems, we see that flink rocksdb checkpoint IO logs are
filling up disk space very very quickly in the order of GB's as the logging
is very verbose. How do we disable or suppress these logs please ? The
rocksdb file checkpoint.cc is dumping huge amount of checkpoint logs
00 or
more tenants to get the better idea about scalability.
Best Regards,
On Mon, 19 Aug 2019 at 16:16, Fabian Hueske wrote:
> Great!
>
> Thanks for the feedback.
>
> Cheers, Fabian
>
> Am Mo., 19. Aug. 2019 um 17:12 Uhr schrieb Ahmad Hassan <
> ahmad.has...@gmail.com&g
; Best, Fabian
>
> Am Do., 15. Aug. 2019 um 19:20 Uhr schrieb Ahmad Hassan <
> ahmad.has...@gmail.com>:
>
>> Hi Fabian,
>>
>> In this case, how do we emit tumbling window when there are no events?
>> Otherwise it’s not possible to emulate a sliding window
hour window.
>
> I don't know your exact business logic, but this is the rough scheme that I
> would follow.
>
> Cheers, Fabian
>
>> Am Fr., 2. Aug. 2019 um 12:25 Uhr schrieb Ahmad Hassan
>> :
>> Hi Fabian,
>>
>> Thanks for this detail. H
hour window.
>
> I don't know your exact business logic, but this is the rough scheme that
> I would follow.
>
> Cheers, Fabian
>
> Am Fr., 2. Aug. 2019 um 12:25 Uhr schrieb Ahmad Hassan <
> ahmad.has...@gmail.com>:
>
>> Hi Fabian,
>>
>> Thanks f
only works for certain aggregation functions. Depending on the
> function, you cannot pre-aggregate which is a hard requirement for this
> approach.
>
> Best, Fabian
>
> Am Do., 1. Aug. 2019 um 20:00 Uhr schrieb Ahmad Hassan <
> ahmad.has...@gmail.com>:
>
>>
Hi Fabian,
> On 4 Jul 2018, at 11:39, Fabian Hueske wrote:
>
> - Pre-aggregate records in a 5 minute Tumbling window. However,
> pre-aggregation does not work for FoldFunctions.
> - Implement the window as a custom ProcessFunction that maintains a state of
> 288 events and aggregates and ret
w function[1] can be help
>
> [1]
> https://ci.apache.org/projects/flink/flink-docs-release-1.8/dev/stream/operators/windows.html#window-functions
> Best,
> Congxian
>
>
> Ahmad Hassan 于2019年7月25日周四 下午5:58写道:
>
>> Hi,
>>
>> We have SlidingProcessin
Hi,
We have SlidingProcessingTimeWindows running with AggregateFunction and
Window Function. How we use MapState within AggregateFunction to keep
storing incoming elements as we receive Millions of elements over 24
running sliding windows ?
If we don't do that then AggregateFunction state grows b
use you declared it as transient field.
>
> Move the declaration inside of "open" function to resolve that
>
> On Mon, Oct 22, 2018 at 3:48 PM Ahmad Hassan
> wrote:
>
>> 2018-10-22 13:46:31,944 INFO org.apache.flink.runtime.taskmanager.Task
>
0)
at org.apache.flink.runtime.taskmanager.Task.run(Task.java:711)
at java.lang.Thread.run(Thread.java:745)
On Sat, 20 Oct 2018 at 11:29, vino yang wrote:
> Hi Ahmad,
>
> Can you try to dump thread info from the Task Manager's JVM instance?
>
> Thanks, vino.
>
> Ahmad Hassan 于20
Flink 1.6.0. Valuestate initialises successful but mapstate hangs
Regards
> On 20 Oct 2018, at 02:55, vino yang wrote:
>
> Hi Ahmad,
>
> Which version of Flink do you use?
>
> Thanks, vino.
>
> Ahmad Hassan 于2018年10月19日周五 下午11:32写道:
>> Hi,
>>
&
Hi,
Initializing mapstate hangs in window function. However if i use valuestate
then it is initialized succcessfully. I am using rocksdb to store the state.
public class MyWindowFunction extends RichWindowFunction
{
private transient MapStateDescriptor productsDescriptor =
new MapStateDescriptor<
Hi Niels,
Can we distinguish within apply function of 'RichWindowFunction' whether it
was called due to onElement trigger call or onProcessingtime trigger call
of a custom Trigger ?
Thanks!
On Wed, 17 Oct 2018 at 12:51, Niels van Kaam wrote:
> Hi Zhen Li,
>
> You can control when a windowed st
Any help/pointers on this please ?
Thanks.
On Thu, 11 Oct 2018 at 10:33, Ahmad Hassan wrote:
> Hi All,
>
> Thanks for the replies. Here is the code snippet of what we want to
> achieve:
>
> We have sliding windows of 24hrs with 5 minutes apart.
>
> inStream
>
This way you could distribute the workload to more parallel
> instances.
>
> Best,
>
> Dawid
>
> On 11/10/18 11:33, Ahmad Hassan wrote:
>
> Hi All,
>
> Thanks for the replies. Here is the code snippet of what we want to
> achieve:
>
> We have sliding window
hat you should do. Did you consider using an AggregateFunction?
>
> Fabian
>
> Am Mi., 10. Okt. 2018 um 11:08 Uhr schrieb Chesnay Schepler <
> ches...@apache.org>:
>
>> In which method are you calling getRuntimeContext()? This method can only
>> be used after open() has been calle
Hi,
We want to use MapState inside fold function to keep the map of all
products that we see in 24 hour window to store huge state in rocksdb
rather than overflowing heap. However, I don't seem to initialise mapstate
within foldfunction or any class that is extending RichMapFunction
private trans
Hi All,
We need two clarifications for using Flink 1.6.0. We have flink jobs
running to handle 100's of tenants with sliding window of 24hrs and slide
by 5 minutes.
1) If checkpointing is enabled and flink job crashes in the middle of
spitting out results to kafka producer. Then if the job resume
:00, Chesnay Schepler wrote:
> Would it be feasible for you to partition your tenants across jobs, like
> for example 100 customers per job?
>
> On 04.07.2018 12:53, Ahmad Hassan wrote:
>
> Hi Fabian,
>
> One job per tenant model soon becomes hard to maintain. For exam
tate
> of 288 events and aggregates and retracts the pre-aggregated records.
>
> Best, Fabian
>
>
> 2018-07-03 15:22 GMT+02:00 Ahmad Hassan :
>
>> Hi Folks,
>>
>> We are using Flink to capture various interactions of a customer with
>> ECommerce store i.e. prod
Hi Folks,
We are using Flink to capture various interactions of a customer with
ECommerce store i.e. product views, orders created. We run 24 hour sliding
window 5 minutes apart which makes 288 parallel windows for a single
Tenant. We implement Fold Method that has various hashmaps to update the
s
d should
> subsume whatever you were previously doing with fold.
> Your previous `WindowStats` class is basically the state accumulator, and
> you need to implement how to update it, merge two accumulators, and
> retrieve the final accumulated result.
>
> For more info, I would point
gt; [1] https://ci.apache.org/projects/flink/flink-docs-
> release-1.3/dev/windows.html#windowfunction---the-generic-case
>
>
> On 29 June 2017 at 5:11:58 PM, Ahmad Hassan (ahmad.has...@gmail.com)
> wrote:
>
> Any thoughts on this problem please?
>
>
> Hi All,
>
&g
Any thoughts on this problem please?
Hi All,
I am collecting millions of events per 24hour for 'N' number of products
where 'N' can be 50k. I use the following fold mechanism with sliding
window:
final DataStream eventStream = inputStream
.keyBy(TENANT, CATEGORY)
.window(SlidingProcessingTimeWi
Hi All,
I am collecting millions of events per hour for 'N' number of products
where 'N' can be 50k. I use the following fold mechanism with sliding
window:
final DataStream eventStream = inputStream
.keyBy(TENANT, CATEGORY)
.window(SlidingProcessingTimeWindows.of(Time.hour(1,Time.minute(5)))
*.f
Thanks Fabian for the advice!
Best Regards,
Dr. Ahmad Hassan
On 23 June 2017 at 09:05, Fabian Hueske wrote:
> Hi Ahmad,
>
> that is not possible, at least not with Flink's built-in windows.
> You can probably implement something like that on top of the DataStream
> API
?
Cheers,
Dr. Ahmad Hassan
Thanks Fabian and Stefan for all the help.
Best Regards,
> On 22 Jun 2017, at 18:06, Fabian Hueske wrote:
>
> 1].
and didn't there not compute anything.
>
>
> 2017-06-22 17:44 GMT+02:00 Ahmad Hassan :
>> Thanks for the answers. My scenario is:
>>
>> | Window A |
>> | Window B |
>>| Window C |
>>
>> If no events are received for Window
eceives a watermark (triggering of a registered
> event-time timer) it should write the buffered records out.
>
> Btw. this only works for event time windows but not for processing time.
>
> Cheers, Fabian
>
> 2017-06-22 16:44 GMT+02:00 Ahmad Hassan :
>
>> Hi Stefan,
>
is Flink’s notion of completeness. This means you can prepare
> those windows and aggregate results and send them downstream to the sink.
>
> Am 22.06.2017 um 15:46 schrieb Ahmad Hassan :
>
> Thanks Stefan, But how the Process function will have these watermarks? I
> have sliding
n end time smaller than the watermark are
> complete. They can then be aggregated and send to the sink.
>
> Best,
> Stefan
>
> > Am 22.06.2017 um 15:15 schrieb Ahmad Hassan :
> >
> > Hi All,
> >
> > I am using categoryID as a keyby attribute for creatin
Hi All,
I am using categoryID as a keyby attribute for creating keyed stream from a
product event stream. Keyed stream then creates time windows for each
category. However, when the window time expires, i want to write the output
data of all the products in all all categories in a single atomic op
37 matches
Mail list logo