Re: Fink: KafkaProducer Data Loss

2017-07-13 Thread Piotr Nowojski
Hi, I’m not sure how relevant is this, but recently I have found and fixed a bug, that in certain conditions was causing data losses for all of the FlinkKafkaProducers in Flink: https://issues.apache.org/jira/browse/FLINK-6996 Namely on

Re: Fink: KafkaProducer Data Loss

2017-07-13 Thread Piotr Nowojski
996 only affects Flink Kafka Producer 010, so I don’t think that’s the > cause here. > > @Ninad > Code to reproduce this would definitely be helpful here, thanks. If you > prefer to provide that privately, that would also be fine. > > Cheers, > Gordon > > On 13 July 2

Re: About KafkaConsumerBase

2017-08-01 Thread Piotr Nowojski
Hi, pendingOffsetsToCommit is a private field which is not accessed from outside of the FlinkKafkaConsumerBase class. It is only used in state manipulation methods, which are not executed in parallel. Thanks, Piotrek > On Aug 1, 2017, at 1:20 PM, aitozi wrote: > >

Re: Kafka 0.11

2017-08-18 Thread Piotr Nowojski
Hi, Yes, Flink Connector for Kafka 0.10 should work without problems with Kafka 0.11. There is also a pending work for a Kafka 0.11 connector that will add support for exactly-once semantic. Piotrek > On Aug 18, 2017, at 5:21 PM, Gabriele Di Bernardo > wrote:

Re: Window + Reduce produces more than 1 output per window

2017-06-19 Thread Piotr Nowojski
Hi, It is difficult for me to respond fully to your question. First of all it would be really useful if you could strip down your example to a minimal version that shows a problem. Unfortunately I was unable to reproduce your issue. I was getting only one output line per window (as expected).

Re: How flink monitor source stream task(Time Trigger) is running?

2017-10-04 Thread Piotr Nowojski
You are welcome :) Piotrek > On Oct 2, 2017, at 1:19 PM, yunfan123 wrote: > > Thank you. > "If SourceFunction.run methods returns without an exception Flink assumes > that it has cleanly shutdown and that there were simply no more elements to > collect/create by

Re: Sink buffering

2017-10-04 Thread Piotr Nowojski
What do you mean by "This always depends on checkpointing interval right?”? In TwoPhaseCommitSinkFunction, transactions are being committed on each Flink checkpoint. I guess same applies to GenericWriteAheadSink. The first one just commits/pre-commits the data on checkpoint, second rewrites

Re: Sink buffering

2017-10-04 Thread Piotr Nowojski
Hi, Do you mean buffer on state and you want to achieve exactly-once HBase sink? If so keep in mind that you will need some kind of transactions support in HBase to make it 100% reliable. Without transactions, buffering messages on state only reduces chance of duplicated records. How much

Re: Avoid duplicate messages while restarting a job for an application upgrade

2017-10-10 Thread Piotr Nowojski
while hundred of thousands were duplicated before. > > I planned to reapply/adapt this patch for the 1.3.2 release when we migrate > to it and maybe later to the 1.4 > > I'm open to suggestion or to help/develop this feature upstream if you want. > > > Le lun. 2 oct. 2017 à 1

Re: Submitting a job via command line

2017-10-13 Thread Piotr Nowojski
Good to hear that :) > On 13 Oct 2017, at 14:40, Alexander Smirnov <asmir...@five9.com> wrote: > > Thank you so much, it helped! > > From: Piotr Nowojski <pi...@data-artisans.com > <mailto:pi...@data-artisans.com>> > Date: Thursday, October 12, 2017 at

Re: Monitoring job w/LocalStreamEnvironment

2017-10-16 Thread Piotr Nowojski
wrote: > > Hi Piotr, > > Thanks for responding, see below. > >> On Oct 12, 2017, at 7:51 AM, Piotr Nowojski <pi...@data-artisans.com >> <mailto:pi...@data-artisans.com>> wrote: >> >> Hi, >> >> Have you read the following doc? >>

Re: Writing to an HDFS file from a Flink stream job

2017-10-12 Thread Piotr Nowojski
nnectors/filesystem_sink.html > > <https://ci.apache.org/projects/flink/flink-docs-release-1.4/dev/connectors/filesystem_sink.html> > > Best, > Aljoscha > >> On 12. Oct 2017, at 14:55, Piotr Nowojski <pi...@data-artisans.com >> <mailto:pi...@data-artisan

Re: Beam Application run on cluster setup (Kafka+Flink)

2017-10-12 Thread Piotr Nowojski
Hi, What do you mean by: > With standalone beam application kafka can receive the message, But in cluster setup it is not working. In your example you are reading the data from Kafka and printing them to console. There doesn’t seems to be anything that writes back to Kafka, so what do you

Re: Monitoring job w/LocalStreamEnvironment

2017-10-12 Thread Piotr Nowojski
Hi, Have you read the following doc? https://ci.apache.org/projects/flink/flink-docs-release-1.4/dev/stream/testing.html There are some hints regarding testing your application. Especially take a look at the

Re: Submitting a job via command line

2017-10-12 Thread Piotr Nowojski
Have you tried this http://mail-archives.apache.org/mod_mbox/flink-user/201705.mbox/%3ccagr9p8bxhljseexwzvxlk+drotyp1yxjy4n4_qgerdzxz8u...@mail.gmail.com%3E

Re: Write each group to its own file

2017-10-12 Thread Piotr Nowojski
Hi, There is no straightforward way to do that. First of all, the error you are getting is because you are trying to start new application ( env.fromElements(items) ) inside your reduce function. To do what you want, you have to hash partition the products based on category (instead of

Re: R/W traffic estimation between Flink and Zookeeper

2017-10-12 Thread Piotr Nowojski
Hi, Are you asking how to measure records/s or is it possible to achieve it? To measure it you can check numRecordsInPerSecond metric. As far if 1000 records/s is possible, it depends on many things like state backend used, state size, complexity of your application, size of the records,

Re: Implement bunch of transformations applied to same source stream in Apache Flink in parallel and combine result

2017-10-12 Thread Piotr Nowojski
Hi, What is the number of events per second that you wish to process? If it’s high enough (~ number of machines * number of cores) you should be just fine, instead of scaling with number of features, scale with number of events. If you have a single data source you still could randomly shuffle

Re: Writing an Integration test for flink-metrics

2017-10-12 Thread Piotr Nowojski
Hi, Doing as you proposed using JMXReporter (or custom reporter) should work. I think there is no easier way to do this at the moment. Piotrek > On 12 Oct 2017, at 04:58, Colin Williams > wrote: > > I have a RichMapFunction and I'd like to ensure Meter

Re: Kafka 11 connector on Flink 1.3

2017-10-12 Thread Piotr Nowojski
Hi, Kafka 0.11 connector depends on some API changes for Flink 1.4, so without rebasing the code and solving some small issues it is not possible to use it for 1.3.x. We are about to finalizing the timeframe for 1.4 release, it would be great if you could come back with this question after

Re: Writing to an HDFS file from a Flink stream job

2017-10-12 Thread Piotr Nowojski
Hi, Maybe this is an access rights issue? Could you try to create and write to same file (same directory) in some other way (manually?), using the same user and the same machine as would Flink job do? Maybe there will be some hint in hdfs logs? Piotrek > On 12 Oct 2017, at 00:19, Isuru

Re: Beam Application run on cluster setup (Kafka+Flink)

2017-10-13 Thread Piotr Nowojski
Hi, What version of Flink are you using. In earlier 1.3.x releases there were some bugs in Kafka Consumer code. Could you change the log level in Flink to debug? Did you check the Kafka logs for some hint maybe? I guess that metrics like bytes read/input records of this Link application are

Re: Writing an Integration test for flink-metrics

2017-10-13 Thread Piotr Nowojski
12.10.2017 14:25, Chesnay Schepler wrote: > You could also write a custom reporter that opens a socket or similar for > communication purposes. > > You can then either query it for the metrics, or even just trigger the > verification in the reporter, > and fail with an error if the

Re: How flink monitor source stream task(Time Trigger) is running?

2017-09-29 Thread Piotr Nowojski
We use Akka's DeathWatch mechanism to detect dead components. TaskManager failure shouldn’t prevent recovering from state (as long as there are enough task slots). I’m not sure if I understand what you mean by "source stream thread" crash. If is was some error during performing a checkpoint so

Re: How flink monitor source stream task(Time Trigger) is running?

2017-09-29 Thread Piotr Nowojski
Any exception thrown by your SourceFunction will be caught by Flink and that will mark a task (that was executing this SourceFuntion) as failed. If you started some custom threads in your SourceFunction, you have to manually propagate their exceptions to the SourceFunction. Piotrek > On Sep

Re: Avoid duplicate messages while restarting a job for an application upgrade

2017-10-02 Thread Piotr Nowojski
tively developed. Piotr Nowojski > On Oct 2, 2017, at 3:35 PM, Antoine Philippot <antoine.philip...@teads.tv> > wrote: > > Hi, > > I'm working on a flink streaming app with a kafka09 to kafka09 use case which > handles around 100k messages per seconds. > > To

Re: Avoid duplicate messages while restarting a job for an application upgrade

2017-10-02 Thread Piotr Nowojski
ature ? > Do you have any hint or details on this part of that "todo list" ? > > > Le lun. 2 oct. 2017 à 16:50, Piotr Nowojski <pi...@data-artisans.com > <mailto:pi...@data-artisans.com>> a écrit : > Hi, > > For failures recovery with Kafka 0.9 it

Re: How flink monitor source stream task(Time Trigger) is running?

2017-09-29 Thread Piotr Nowojski
I am still not sure what do you mean by “thread crash without throw”. If SourceFunction.run methods returns without an exception Flink assumes that it has cleanly shutdown and that there were simply no more elements to collect/create by this task. If it continue working, without throwing an

Re: starting query server when running flink embedded

2017-09-29 Thread Piotr Nowojski
Hi, You can take a look at how is it done in the exercises here . There are example solutions that run on a local environment. I Hope that helps :) Piotrek > On Sep 28, 2017, at 11:22 PM, Henri Heiskanen

Re: state of parallel jobs when one task fails

2017-09-29 Thread Piotr Nowojski
Hi, Yes, by default Flink will restart all of the tasks. I think that since Flink 1.3, you can configure a FailoverStrategy to change this behavior.

Re: Very low-latency - is it possible?

2017-08-31 Thread Piotr Nowojski
Achieving 1ms in any distributed system might be problematic, because even simplest ping messages between worker nodes take ~0.2ms. However, as you stated your desired throughput (40k records/s) and state is small, so maybe there is no need for using a distributed system for that? You could

Re: Bucketing/Rolling Sink: New timestamp appeded to the part file name everytime a new part file is rolled

2017-09-01 Thread Piotr Nowojski
Hi, BucketingSink doesn’t support the feature that you are requesting, you can not specify a dynamically generated prefix/suffix. Piotrek > On Aug 31, 2017, at 7:12 PM, Raja.Aravapalli > wrote: > > > Hi, > > I have a flink application that is streaming data

Re: Modify field topics (KafkaConsumer) during runtime

2017-08-31 Thread Piotr Nowojski
Hi, As far as I know it is not possible to do it on the fly. There is planned feature for discovering topics using some regex: https://github.com/apache/flink/pull/3746 https://issues.apache.org/jira/browse/FLINK-5704

Re: SLF4j logging system gets clobbered?

2017-10-19 Thread Piotr Nowojski
Hi, What versions of Flink/logback are you using? Have you read this: https://ci.apache.org/projects/flink/flink-docs-release-1.3/dev/best_practices.html#use-logback-when-running-flink-out-of-the-ide--from-a-java-application

Re: Set heap size

2017-10-19 Thread Piotr Nowojski
Hi, Just log into the machine and check it’s memory consumption using htop or a similar tool under the load. Remember about subtracting Flink’s memory usage and and file system cache. Piotrek > On 19 Oct 2017, at 10:15, AndreaKinn wrote: > > About task manager heap size

Re: HBase config settings go missing within Yarn.

2017-10-20 Thread Piotr Nowojski
Hi, What do you mean by saying: > When I open the logfiles on the Hadoop cluster I see this: The error doesn’t come from Flink? Where do you execute hbaseConfig.addResource(new Path("file:/etc/hbase/conf/hbase-site.xml")); ? To me it seems like it is a problem with misconfigured HBase and

Re: Does heap memory gets released after computing windows?

2017-10-20 Thread Piotr Nowojski
Hi, Memory used by session windows should be released once window is triggered (allowedLateness can prolong window’s life). Unless your code introduces some memory leak (by not releasing references) everything should be garbage collected. Keep in mind that session windows with time gap of 10

Re: Flink BucketingSink, subscribe on moving of file into final state

2017-10-20 Thread Piotr Nowojski
functionality, to handle moments of moving the file into final state. > I thought, that maybe someone has already implemented such thing or knows any > other approaches that will help me to not copy/ paste existing sink impl )) > > Thx ! > > >> On 20 Oct 2017, at 14:37, Piotr

Re: Flink flick cancel vs stop

2017-10-24 Thread Piotr Nowojski
I would propose implementations of NewSource to be not blocking/asynchronous. For example something like public abstract Future getCurrent(); Which would allow us to perform some certain actions while there are no data available to process (for example flush output buffers). Something like

Re: Flink memory leak

2017-11-10 Thread Piotr Nowojski
jobmanager1.log and taskmanager2.log are the same. Can you also submit files containing std output? Piotrek > On 10 Nov 2017, at 09:35, ÇETİNKAYA EBRU ÇETİNKAYA EBRU > <b20926...@cs.hacettepe.edu.tr> wrote: > > On 2017-11-10 11:04, Piotr Nowojski wrote: >> Hi, >>

Re: How & Where does flink stores data for aggregations.

2017-11-24 Thread Piotr Nowojski
Hi, Flink will have to maintain state of the defined aggregations per each window and key (the more names you have, the bigger the state). Flink’s state backend will be used for that (for example memory or rocksdb). However in most cases state will be small and not dependent on the length of

Re: Off heap memory issue

2017-12-13 Thread Piotr Nowojski
er has 383,120 > instances. I don't know if this could be related. > > I hope that we can test this soon, and will let you know if this fixed the > problem. > > Thanks. > > > On 15 November 2017 at 13:18, Piotr Nowojski <pi...@data-artisans.com > <mailto:p

Re: Flink flick cancel vs stop

2017-12-14 Thread Piotr Nowojski
a > savepoint will not wait for the Kafka transactions to bet committed and is > not atomic. > > So it seems like there is a need for an atomic stop or cancel with savepoint > that waits for transactional sinks to commit and then immediately stop any > further message processing. > &

Re: Custom Metrics

2017-12-14 Thread Piotr Nowojski
.addGroup("Users") > .meter(userId, new DropwizardMeterWrapper(new > com.codahale.metrics.Meter())); > Thanks a bunch. > > On Mon, Dec 11, 2017 at 11:12 PM, Piotr Nowojski <pi...@data-artisans.com > <mailto:pi...@data-artisans.com>> wrote: > Hi, >

Re: Flink memory leak

2017-11-17 Thread Piotr Nowojski
task managers memory pool with VisualVm and Console, we > attached the screenshot of results. > Could you help us about evaluating the results? > > -Ebru > > >> On 14 Nov 2017, at 19:29, Piotr Nowojski <pi...@data-artisans.com >> <mailto:pi...@data-artisans.com&

Re: org.apache.flink.runtime.io.network.NetworkEnvironment causing memory leak?

2017-11-17 Thread Piotr Nowojski
Hi, If the TM is not responding check the TM logs if there is some long gap in logs. There might be three main reasons for such gaps: 1. Machine is swapping - setup/configure your machine/processes that machine never swap (best to disable swap altogether) 2. Long GC full stops - look how to

Re: Flink memory leak

2017-11-10 Thread Piotr Nowojski
lt;b20926...@cs.hacettepe.edu.tr> wrote: > > On 2017-11-10 13:14, Piotr Nowojski wrote: >> jobmanager1.log and taskmanager2.log are the same. Can you also submit >> files containing std output? >> Piotrek >>> On 10 Nov 2017, at 09:35, ÇETİNKAYA EBRU ÇETİNKAYA EBRU >>&

Re: Flink memory leak

2017-11-10 Thread Piotr Nowojski
-pnowojski-taskmanager-9-piotr-mbp.log Description: Binary data > On 10 Nov 2017, at 16:05, ÇETİNKAYA EBRU ÇETİNKAYA EBRU > <b20926...@cs.hacettepe.edu.tr> wrote: > > On 2017-11-10 18:01, ÇETİNKAYA EBRU ÇETİNKAYA EBRU wrote: >> On 2017-11-10 17:50, Piotr Nowojski wrote: &

Re: Correlation between data streams/operators and threads

2017-11-14 Thread Piotr Nowojski
t; attached a snap of my job plan visualization. > > 3. Have attached the logs and exception raised (15min - configured akka > timeout) after submitting the job. > > Thanks, > Shailesh > > > On Tue, Nov 14, 2017 at 2:46 PM, Piotr Nowojski <pi...@data-artisans.com

Re: Correlation between data streams/operators and threads

2017-11-13 Thread Piotr Nowojski
lesh > > On Fri, Nov 10, 2017 at 5:52 PM, Piotr Nowojski <pi...@data-artisans.com > <mailto:pi...@data-artisans.com>> wrote: > 1. It’s a little bit more complicated then that. Each operator chain/task > will be executed in separate thread (parallelism > Multiplies

Re: Correlation between data streams/operators and threads

2017-11-14 Thread Piotr Nowojski
ave other questions or encounter some issues. > > Thanks, Piotrek > > >> On 13 Nov 2017, at 14:49, Shailesh Jain <shailesh.j...@stellapps.com >> <mailto:shailesh.j...@stellapps.com>> wrote: >> >> Thanks, Piotr. I'll try it out and will get back in case of

Re: Flink memory leak

2017-11-14 Thread Piotr Nowojski
gt; > On Tue, Nov 14, 2017 at 4:02 PM, Piotr Nowojski <pi...@data-artisans.com > <mailto:pi...@data-artisans.com>> wrote: > Ebru, Javier, Flavio: > > I tried to reproduce memory leak by submitting a job, that was generating > classes with random names. And ind

Re: Flink memory leak

2017-11-14 Thread Piotr Nowojski
encing? If not, I would really need more help/information from you to track this down. Piotrek > On 10 Nov 2017, at 15:12, ÇETİNKAYA EBRU ÇETİNKAYA EBRU > <b20926...@cs.hacettepe.edu.tr> wrote: > > On 2017-11-10 13:14, Piotr Nowojski wrote: >> jobmanager1.log and taskmanager2.

Re: Correlation between data streams/operators and threads

2017-11-21 Thread Piotr Nowojski
I'm not sure how this is different from the current > approach I had provided the code for above, and will it solve this problem of > different data streams not getting distributed across slots? > > Thanks again, > Shailesh > > On Fri, Nov 17, 2017 at 3:01 PM, P

Re: Off heap memory issue

2017-11-15 Thread Piotr Nowojski
Hi, I have been able to observe some off heap memory “issues” by submitting Kafka job provided by Javier Lopez (in different mailing thread). TL;DR; There was no memory leak, just memory pool “Metaspace” and “Compressed Class Space” are growing in size over time and are only rarely garbage

Re: Docker-Flink Project: TaskManagers can't talk to JobManager if they are on different nodes

2017-11-06 Thread Piotr Nowojski
ssor.processInput(StreamInputProcessor.java:213) > at > org.apache.flink.streaming.runtime.tasks.OneInputStreamTask.run(OneInputStreamTask.java:69) > at > org.apache.flink.streaming.runtime.tasks.StreamTask.invoke(StreamTask.java:263) > at org.apache.flink.runtime.taskm

Re: How to best create a bounded session window ?

2017-11-09 Thread Piotr Nowojski
It might be more complicated if you want to take into account events coming in out of order. For example you limit length of window to 5 and you get the following events: 1 2 3 4 6 7 8 5 Do you want to emit windows: [1 2 3 4 5] (length limit exceeded) + [6 7 8] ? Or are you fine with

Re: Correlation between data streams/operators and threads

2017-11-09 Thread Piotr Nowojski
Hi, 1. https://ci.apache.org/projects/flink/flink-docs-release-1.3/dev/parallel.html Number of threads executing would be roughly speaking equal to of the number of input data streams multiplied by the

Re: Weird performance on custom Hashjoin w.r.t. parallelism

2017-11-09 Thread Piotr Nowojski
Hi, Yes as you correctly analysed parallelism 1 was causing problems, because it meant that all of the records must been gathered over the network from all of the task managers. Keep in mind that even if you increase parallelism to “p”, every change in parallelism can slow down your

Re: How to best create a bounded session window ?

2017-11-09 Thread Piotr Nowojski
n the > leading boundary is forever increasing. We may decide to fire_and_purge. fire > etc but the window remains live. Or did I get that part wrong ? > > > Vishal. > > > > > On Thu, Nov 9, 2017 at 8:24 AM, Piotr Nowojski <pi...@data-artisans.com >

Re: Flink memory leak

2017-11-09 Thread Piotr Nowojski
; > On 2017-11-08 18:30, Piotr Nowojski wrote: >> Btw, Ebru: >> I don’t agree that the main suspect is NetworkBufferPool. On your >> screenshots it’s memory consumption was reasonable and stable: 596MB >> -> 602MB -> 597MB. >> PoolThreadCache memory usage

Re: Flink memory leak

2017-11-10 Thread Piotr Nowojski
BRU > <b20926...@cs.hacettepe.edu.tr> wrote: > > On 2017-11-09 20:08, Piotr Nowojski wrote: >> Hi, >> Could you attach full logs from those task managers? At first glance I >> don’t see a connection between those exceptions and any memory issue >> that you m

Re: Flink memory leak

2017-11-08 Thread Piotr Nowojski
Hi Ebru and Javier, Yes, if you could share this example job it would be helpful. Ebru: could you explain in a little more details how does your Job(s) look like? Could you post some code? If you are just using maps and filters there shouldn’t be any network transfers involved, aside from

Re: Flink memory leak

2017-11-08 Thread Piotr Nowojski
> "TM_MAX_OFFHEAP_SIZE="2G"; and nothing seems to work. > > Thanks for your help. > > On 8 November 2017 at 13:26, ÇETİNKAYA EBRU ÇETİNKAYA EBRU > <b20926...@cs.hacettepe.edu.tr <mailto:b20926...@cs.hacettepe.edu.tr>> wrote: > On 2017-11-08 15:20, Piotr Nowo

Re: Flink memory leak

2017-11-08 Thread Piotr Nowojski
heap size. Piotrek > On 8 Nov 2017, at 15:28, Javier Lopez <javier.lo...@zalando.de> wrote: > > Yes, I tested with just printing the stream. But it could take a lot of time > to fail. > > On Wednesday, 8 November 2017, Piotr Nowojski <pi...@data-artisans.com >

Re: Flink memory leak

2017-11-08 Thread Piotr Nowojski
need data. With data it will die faster. I tested as well with a > small data set, using the fromElements source, but it will take some time to > die. It's better with some data. > > On 8 November 2017 at 14:54, Piotr Nowojski <pi...@data-artisans.com > <mailto:pi...@da

Re: Flink memory leak

2017-11-08 Thread Piotr Nowojski
ailto:javier.lo...@zalando.de>> wrote: >> >> Yes, I tested with just printing the stream. But it could take a lot of time >> to fail. >> >> On Wednesday, 8 November 2017, Piotr Nowojski <pi...@data-artisans.com >> <mailto:pi...@data-artisans.com>

Re: Docker-Flink Project: TaskManagers can't talk to JobManager if they are on different nodes

2017-11-02 Thread Piotr Nowojski
Did you try to expose required ports that are listed in the README when starting the containers? https://github.com/apache/flink/tree/master/flink-contrib/docker-flink Ports: • The Web Client is on port 48081

Re: REST api: how to upload jar?

2017-12-11 Thread Piotr Nowojski
Hi, Have you tried this https://stackoverflow.com/questions/41724269/apache-flink-rest-client-jar-upload-not-working ? Piotrek > On 11 Dec 2017, at 14:22, Edward wrote: > >

Re: Custom Metrics

2017-12-11 Thread Piotr Nowojski
Hi, I’m not sure if I completely understand your issue. 1. - You don’t have to pass RuntimeContext, you can always pass just the MetricGroup or ask your components/subclasses “what metrics do you want to register” and register them at the top level. - Reporting tens/hundreds/thousands of

Re: Exception when using the time attribute in table API

2017-12-11 Thread Piotr Nowojski
Hi, NoSuchMethodError probably comes from some mismatched compile/runtime versions of the Flink. Do you have to use 1.4-SNAPSHOT version? It can change on daily basis, so you have to be more careful about what Flink jar's you are using at runtime and what on compile time. If you really need

Re: ayncIO & TM akka response

2017-12-11 Thread Piotr Nowojski
Hi, Please search the task manager logs for the potential reason of failure/disconnecting around the time when you got this error on the job manager. There should be some clearly visible exception. Thanks, Piotrek > On 9 Dec 2017, at 20:35, Chen Qin wrote: > > Hi there,

Re: Custom Metrics

2017-12-11 Thread Piotr Nowojski
x and grafana to plot the metrics. > > The option 2 that I had in mind was to collect all metrics and use influx db > sink to report it directly inside the pipeline. But it seems reporting per > node might not be possible. > > > On Mon, Dec 11, 2017 at 3:14 AM, Piotr N

Re: How to deal with dynamic types

2017-12-11 Thread Piotr Nowojski
Hi, For truly dynamic class you would need a custom TypeInformation or TypeDeserializationSchema and store the fields on some kind of Map. Maybe something could be done with inheritance if records that always share the same fields could be deserialized to some specific class

Re: Metric reporters with non-static ports

2017-12-21 Thread Piotr Nowojski
I am not sure (and because of holiday season you might not get an answer quickly), however I do not see a way to obtain this port other then by looking into the log files. On the other hand, I have an impression, that intention of this feature was that if you must execute N reporters on one

Re: entrypoint for executing job in task manager

2017-12-21 Thread Piotr Nowojski
Open method is called just before any elements are processed. You can hook in any initialisation logic there, including initialisation of a static context. However keep in mind, that since this context is static, it will be shared between multiple operators (if you are running parallelism >

Re: This server is not the leader for that topic-partition

2018-05-05 Thread Piotr Nowojski
t; final Properties props = new Properties(); > ... > props.put(ProducerConfig.RETRIES_CONFIG, 10); > props.put(ConsumerConfig.MAX_POLL_INTERVAL_MS_CONFIG, > Integer.toString(Integer.MAX_VALUE)); > props.put(ConsumerConfig.SESSION_TIMEOUT_MS_CONFIG, 2); > > > > On

Re: This server is not the leader for that topic-partition

2018-05-07 Thread Piotr Nowojski
.x. Piotrek > On 7 May 2018, at 12:12, Alexander Smirnov <alexander.smirn...@gmail.com> > wrote: > > Hi Piotr, using 0.11.0 Kafka version > > On Sat, May 5, 2018 at 10:19 AM Piotr Nowojski <pi...@data-artisans.com > <mailto:pi...@data-artisans.com>> wr

Re: intentional back-pressure (or a poor man's side-input)

2018-05-04 Thread Piotr Nowojski
s this initial state load in open()... not sure how > to work out the keying. > > I'll post back if I get anywhere with these ideas. > > On 5/3/18 10:49 AM, Piotr Nowojski wrote: >> Maybe it could work with Flink’s 1.5 credit base flow control. But you

Re: How to broadcast messages to all task manager instances in cluster?

2018-05-14 Thread Piotr Nowojski
> message is broadcast to each task manager, the data streams will see the > change. The logic in data streams is quite simple, just get some parameters > from the static variable. So I think to add connect and flatmap to each of > them is too verbose. I am wondering is there any better

Re: Checkpoint is not triggering as per configuration

2018-05-15 Thread Piotr Nowojski
n skip the checkpoint of "Source: Custom File Source" but still > having checkpoint on "Split Reader: Custom File Source"? > Thanks, > Tao > > On Fri, May 11, 2018 at 4:34 AM, Piotr Nowojski <pi...@data-artisans.com > <mailto:pi...@data-artisans.com>>

Re: Checkpoint is not triggering as per configuration

2018-05-11 Thread Piotr Nowojski
Hi, It’s not considered as a bug, only a missing not yet implemented feature (check my previous responses for the Jira ticket). Generally speaking using file input stream for DataStream programs is not very popular, thus this was so far low on our priority list. Piotrek > On 10 May 2018, at

Re: How to broadcast messages to all task manager instances in cluster?

2018-05-11 Thread Piotr Nowojski
Hi, I don’t quite understand your problem. If you broadcast message as an input to your operator that depends on this configuration, each instance of your operator will receive this configuration. It shouldn't matter whether Flink scheduled your operator on one, some or all of the

Re: Late data before window end is even close

2018-05-11 Thread Piotr Nowojski
ebug futher what's causing late data on a production > stream job, please let me know. > > On Fri, May 11, 2018 at 2:18 PM, Piotr Nowojski <pi...@data-artisans.com > <mailto:pi...@data-artisans.com>> wrote: > Hey, > > Actually I think Fabian initial messag

Re: Late data before window end is even close

2018-05-11 Thread Piotr Nowojski
Hey, Actually I think Fabian initial message was incorrect. As far as I can see in the code of WindowOperator (last lines of org.apache.flink.streaming.runtime.operators.windowing.WindowOperator#processElement ), the element is sent to late side output if it is late AND it wasn’t assigned to

Re: Flink Forward SF 2018 Videos

2018-05-11 Thread Piotr Nowojski
Hi, Previous videos were always uploaded there, so I guess the new one should appear there shortly. Laura might now something more about it. Thanks, Piotrek > On 10 May 2018, at 23:44, Rafi Aroch wrote: > > Hi, > > Are there any plans to upload the videos to the Flink

Re: PartitionNotFoundException after deployment

2018-05-04 Thread Piotr Nowojski
Ufuk: I don’t know why. +1 for your other suggestions. Piotrek > On 4 May 2018, at 14:52, Ufuk Celebi wrote: > > Hey Gyula! > > I'm including Piotr and Nico (cc'd) who have worked on the network > stack in the last releases. > > Registering the network structures

Re: Question about the behavior of TM when it lost the zookeeper client session in HA mode

2018-05-15 Thread Piotr Nowojski
Hi, It looks like there was an error in asynchronous job of sending the records to Kafka. Probably this is a collateral damage of loosing connection to zookeeper. Piotrek > On 15 May 2018, at 13:33, Ufuk Celebi wrote: > > Hey Tony, > > thanks for the detailed report. > >

Re: Decrease initial source read speed

2018-05-23 Thread Piotr Nowojski
Hi, Yes, Flink 1.5.0 will come with better tools to handle this problem. Namely you will be able to limit the “in flight” data, by controlling the number of assigned credits per channel/input gate. Even without any configuring Flink 1.5.0 will out of the box buffer less data, thus mitigating

Re: Decrease initial source read speed

2018-05-23 Thread Piotr Nowojski
n out the imbalance of the assumption of record size being proportional to number of bytes. Piotrek > On 23 May 2018, at 15:40, Piotr Nowojski <pi...@data-artisans.com> wrote: > > Hi, > > Yes if you have mixed workload in your pipeline, it is matter of finding a

Re: Decrease initial source read speed

2018-05-23 Thread Piotr Nowojski
sed data later. > Changing buffer sizes globally will probably affect throughput of later > steps, as you wrote. > > >> On 23 May 2018, at 14:48, Piotr Nowojski <pi...@data-artisans.com >> <mailto:pi...@data-artisans.com>> wrote: >> >> Hi, >> >&g

Re: FlinkKinesisProducer weird behaviour

2018-05-24 Thread Piotr Nowojski
Hi, Have you tried to write the same records, with exactly the same configuration to the Kinesis, but outside of Flink (with some standalone Java application)? Piotrek > On 24 May 2018, at 09:40, Rafi Aroch wrote: > > Hi, > > We're using Kinesis as our input & output

Re: Starting beam pipeline from savepoint

2018-05-24 Thread Piotr Nowojski
Hi, I am not sure, but it probably brings down to the question, whether you can restore from a save point when using RemoteEnvironment - and answer for this question is unfortunately no. Maybe Aljoscha will know something more. Piotrek > On 23 May 2018, at 19:25, borisbolvig

Re: Akka Http used in custom RichSourceFunction

2018-05-25 Thread Piotr Nowojski
ironment.scala:629) > at net.vankaam.flink.WebSocketSample$.main(WebSocketSample.scala:42) > at net.vankaam.flink.WebSocketSample.main(WebSocketSample.scala) > > > Cheers, > Niels > > > On Thu, May 24, 2018 at 4:08 PM Piotr Nowojski <pi...@data-artisans.com &

Re: Akka Http used in custom RichSourceFunction

2018-05-25 Thread Piotr Nowojski
bsocketclient> > > Niels > > > > On Fri, May 25, 2018 at 11:00 AM Piotr Nowojski <pi...@data-artisans.com > <mailto:pi...@data-artisans.com>> wrote: > Hi, > > Yes, this might be the cause of the issue, because indeed it looks like your >

Re: When is 1.5 coming out

2018-05-24 Thread Piotr Nowojski
It seems like Flink-9349 will make it to the 1.5, because RC5 was cancelled. Piotrek > On 24 May 2018, at 14:03, Vishal Santoshi wrote: > > Flink-9349 can we out this to 1.5 release. > > On Thu, May 24, 2018, 7:59 AM Vishal Santoshi

Re: When is 1.5 coming out

2018-05-24 Thread Piotr Nowojski
; Thanks. Do we have a timeline for when we would be out with 1.5. It seems got > delayed again ? > > On Thu, May 24, 2018 at 8:07 AM, Piotr Nowojski <pi...@data-artisans.com > <mailto:pi...@data-artisans.com>> wrote: > It seems like Flink-9349 will make it to the 1.

Re: Akka Http used in custom RichSourceFunction

2018-05-24 Thread Piotr Nowojski
Hi, Please take a look on https://ci.apache.org/projects/flink/flink-docs-release-1.4/monitoring/debugging_classloading.html . Especially check if you are using child-first class loading

Re: FlinkKinesisProducer weird behaviour

2018-05-24 Thread Piotr Nowojski
is-kpl-consumer-deaggregation.html > > <https://docs.aws.amazon.com/streams/latest/dev/kinesis-kpl-consumer-deaggregation.html> > [2] > https://docs.aws.amazon.com/streams/latest/dev/kinesis-kpl-integration.html > <https://docs.aws.amazon.com/streams/latest/dev/kinesis-kpl-integ

Re: Multiple stream operator watermark handling

2018-05-24 Thread Piotr Nowojski
Hi, From top of my head I can imagine two solutions: 1. Override the default behaviour of the operator via for example org.apache.flink.streaming.api.datastream.ConnectedStreams#transform 2. Can you set control stream’s watermark to Watermark#MAX_WATERMARK or maybe Watermark#MAX_WATERMARK - 1

  1   2   3   4   5   6   7   >