Re: Memory Issue

2017-08-24 Thread Govindarajan Srinivasaraghavan
the checkpointing configuration. > > Using RocksDB as the state backend, Flink will have some objects on the heap, > like timers (we will move them to RocksDB as well in the near future) but the > majority will be off heap. > > Stephan > > >> On Thu, Aug 24, 2017 at 5:28 AM,

Re: Memory Issue

2017-08-23 Thread Govindarajan Srinivasaraghavan
nerally you should avoid creating PoJo/objects as much as possible in a > long running Streaming job. > > > > > On 21. Aug 2017, at 05:29, Govindarajan Srinivasaraghavan < > govindragh...@gmail.com> wrote: > > > > Hi, > > > > I have a pipeline running

Memory Issue

2017-08-20 Thread Govindarajan Srinivasaraghavan
Hi, I have a pipeline running on flink which ingests around 6k messages per second. Each message is around 1kb and it passes through various stages like filter, 5 sec tumbling window per key etc.. and finally flatmap to computation before sending it to kafka sink. The data is first ingested as pro

Task slot data

2017-08-08 Thread Govindarajan Srinivasaraghavan
Hi, I have a complex algorithm running on a flatmap operator and it requires cache and metrics instances to be passed to various classes. Since the task slots run on the same JVM, I'm not sure if I will be able to use static members or singleton classes. Can I use ThreadLocal variables to isolate

Re: Eventime window

2017-08-02 Thread Govindarajan Srinivasaraghavan
e results after 10 seconds wall-clock/processing-time. So >> either you use a processing-time window or you implement a custom trigger >> that triggers both on event-time or on a timer that you have set after 10 s >> processing-time. >> >> Timo >> >> >

Re: Eventime window

2017-08-02 Thread Govindarajan Srinivasaraghavan
window comes with some > overhead. So you should choose your memory/state backends and parallelism > accordingly. > > Hope that helps. > > Timo > > >> Am 02.08.17 um 06:54 schrieb Govindarajan Srinivasaraghavan: >> Hi, >> >> I have few que

Eventime window

2017-08-01 Thread Govindarajan Srinivasaraghavan
Hi, I have few questions regarding event time windowing. My scenario is devices from various timezones will send messages with timestamp and I need to create a window per device for 10 seconds. The messages will mostly arrive in order. Here is my sample code to perform windowing and aggregating t

KeyBy State

2017-07-31 Thread Govindarajan Srinivasaraghavan
Hi, I have a keyby state but the key can change quite frequently for the same user and I need the previous keyBy state value for the user if there is a key change. Right now I'm using redis cache for the global state. Is there a way to achieve this within flink?

Connect more than two streams

2017-07-24 Thread Govindarajan Srinivasaraghavan
Hi, I have two streams reading from kafka, one for data and other for control. The data stream is split by type and there are around six types. Each type has its own processing logic and finally everything has to be merged to get the collective state per device. I was thinking I could connect mul

Re: Streaming Exception

2017-03-10 Thread Govindarajan Srinivasaraghavan
f are also often helpful. > > > > On Fri, Mar 10, 2017 at 10:46 AM, Kaibo Zhou wrote: > >> I think this is not the root cause of job failure, this task is caused by >> other tasks failing. You can check the log of the first failed task. >> >> 2017-03-10 12:25 GMT+08

Streaming Exception

2017-03-09 Thread Govindarajan Srinivasaraghavan
Hi All, I see the below error after running my streaming job for a while and when the load increases. After a while the task manager becomes completely dead and the job keeps on restarting. Also when I checked if there is an back pressure in the UI, it kept on saying sampling in progress and no r

Re: ElasticsearchSink Exception

2017-02-28 Thread Govindarajan Srinivasaraghavan
be causing this? I'm asking because you recently worked on the > Elasticsearch connectors, right? > > On Sun, 26 Feb 2017 at 04:26 Govindarajan Srinivasaraghavan < > govindragh...@gmail.com> wrote: > >> Thanks Flavio. I tried with multiple versions but still the same except

Re: ElasticsearchSink Exception

2017-02-25 Thread Govindarajan Srinivasaraghavan
1.2.0' compile group: 'org.apache.flink', name: 'flink-clients_2.10', version: '1.2.0' compile group: 'org.apache.flink', name: 'flink-connector-elasticsearch2_2.10', version: '1.2.0' On Sat, Feb 25, 2017 at 1:26 AM, Flavio Pomperma

ElasticsearchSink Exception

2017-02-24 Thread Govindarajan Srinivasaraghavan
Hi All, I'm getting the below exception when I start my flink job. I have verified the elastic search host and it seems to be working well. I have also tried including the below dependecies to my project but nothing works. Need some help. Thanks. compile group: 'org.apache.lucene', name: 'lucene-

Flink Job Exception

2017-02-15 Thread Govindarajan Srinivasaraghavan
Hi All, I'm trying to run a streaming job with flink 1.2 version and there are 3 task managers with 12 task slots. Irrespective of the parallelism that I give it always fails with the below error and I found a JIRA link corresponding to this issue. Can I know by when this will be resolved since I'

Re: Dynamic Scaling

2016-12-26 Thread Govindarajan Srinivasaraghavan
Hi All, It would great if someone can help me with my questions. Appreciate all the help. Thanks. > On Dec 23, 2016, at 12:11 PM, Govindarajan Srinivasaraghavan > wrote: > > Hi, > > We have a computation heavy streaming flink job which will be processing > around 1

Dynamic Scaling

2016-12-23 Thread Govindarajan Srinivasaraghavan
Hi, We have a computation heavy streaming flink job which will be processing around 100 million message at peak time and around 1 million messages in non peak time. We need the capability to dynamically scale so that the computation operator can scale up and down during high or low work loads resp

Stream Iterations

2016-12-18 Thread Govindarajan Srinivasaraghavan
Hi All, I have a use case for which I need some suggestions. It's a streaming application with kafka source and then groupBy, keyBy and perform some calculations. The output of each calculation has to be a side input for the next calculation and also it needs to be sent to a sink. Right now I'm a

Flink Metrics

2016-10-16 Thread Govindarajan Srinivasaraghavan
Hi, I am currently using flink 1.2 snapshot and instrumenting my pipeline with flink metrics. One small suggestion I have is currently the Meter interface only supports getRate() which is always the one minute rate. It would great if all the rates (1 min, 5 min & 15 min) are exposed to get a bett

Re: Using Flink

2016-10-03 Thread Govindarajan Srinivasaraghavan
ing > behaviour according to this stream. > > - Can I pass on state between operators? If I need the state stored on > previous operators, how can I fetch it? > > I don’t think this is possible. > > > Best Regards, > Gordon > > > On October 3, 2016 at 2

Using Flink

2016-10-02 Thread Govindarajan Srinivasaraghavan
Hi, I have few questions on how I need to model my use case in flink. Please advise. Thanks for the help. - I'm currently running my flink program on 1.2 SNAPSHOT with kafka source and I have checkpoint enabled. When I look at the consumer offsets in kafka it appears to be stagnant and there

Re: Flink: How to handle external app configuration changes in flink

2016-09-26 Thread Govindarajan Srinivasaraghavan
the same state. For example: > > val dataStream = env.addSource(dataSource).keyBy("userId")val > filterStream = env.addSource(filterSource).broadcast() > val connectedStream = dataStream > .connect(filterStream) > .flatMap(yourFilterFunction) > > ​ > I hop

Flink: How to handle external app configuration changes in flink

2016-09-25 Thread Govindarajan Srinivasaraghavan
Hi, My requirement is to stream millions of records in a day and it has huge dependency on external configuration parameters. For example, a user can go and change the required setting anytime in the web application and after the change is made, the streaming has to happen with the new application

Questions on flink

2016-09-24 Thread Govindarajan Srinivasaraghavan
Hi, I'm working on apache flink for data streaming and I have few questions. Any help is greatly appreciated. Thanks. 1) Are there any restrictions on creating tumbling windows. For example, if I want to create a tumbling window per user id for 2 secs and let’s say if I have more than 10 million