Re: Kafka Producer - Null Pointer Exception when processing by element

2017-07-17 Thread earellano
Tzu-Li (Gordon) Tai wrote > Basically, when two operators are chained together, the output of the > first operator is immediately chained to the processElement of the next > operator; it’s therefore just a consecutive invocation of processElements > on the chained operators. There will be no thread

Re: Flink Elasticsearch Connector: Lucene Error message

2017-07-17 Thread Fabian Wollert
TL;DR: remove all lucene and elasticsearch libs in your flink env and just use maven to manage dependencies, when working with the flink elasticsearch connector. so in the first place i deleted the libs in the folder to see if its working, but it did not. then we thought if maybe flink loads alrea

Re: FileNotFoundException when restoring checkpoint

2017-07-17 Thread Stefan Richter
After giving it a second thought, this problem could a side effect of the issue fixed in https://issues.apache.org/jira/browse/FLINK-6964 . If you want, you can try if your problem is fixed in the latest master. This fix will also go into the 1.

Re: FileNotFoundException when restoring checkpoint

2017-07-17 Thread Stefan Richter
Hi, You assumed correctly that savepoints are always self-contained. Are you using externalized checkpoints? There is a known problem in that was fixed in the latest master and will go into 1.3.2, but this might be a different problem. You are also correct that incremental checkpoints can refer

Re: FileNotFoundException when restoring checkpoint

2017-07-17 Thread Chesnay Schepler
Hello, If i recall correctly savepoints are always self-contained even if incremental checkpointing is enabled. However, this doesn't appear to be documented anywhere. As for the missing file, I'm looping in Stefan who is more knowledgeable about incremental checkpointing (and potentially kno

FileNotFoundException when restoring checkpoint

2017-07-17 Thread Shai Kaplan
Hi. I'm running Flink 1.3.1 with checkpoints stored in Azure blobs. Incremental checkpoints feature is on. The job is trying to restore a checkpoint and consistently gets: java.lang.IllegalStateException: Could not initialize keyed state backend. at org.apache.flink.streaming.api.operato

Re: Latency Measurement

2017-07-17 Thread Chesnay Schepler
Hello, As for 1), my suspicion is that this is caused by chaining. If the map function is chained to the kafka source then the latency markers are always immediately forwarded, regardless of what your map function is doing. If the map function is indeed chained to the source, could you try agai

Latency Measurement

2017-07-17 Thread Paolo Cristofanelli
Hi, I would like to understand how to measure the latency of a record. I have set up a simple project with a Kafka consumer that reads from a topic and performs a simple map (with a thread sleep inside). In order to measure the latency of this mapper I have added env.getConfig().setLatencyTrackin

Re: Reading static data

2017-07-17 Thread Fabian Hueske
You could either use a broadcast variable [1] or the distributed cache [2]. Best, Fabian [1] https://ci.apache.org/projects/flink/flink-docs-release-1.3/dev/batch/index.html#broadcast-variables [2] https://ci.apache.org/projects/flink/flink-docs-release-1.3/dev/batch/index.html#distributed-cache

Re: Flink Elasticsearch Connector: Lucene Error message

2017-07-17 Thread Tzu-Li (Gordon) Tai
Hi, I would also recommend checking the `lib/` folder of your Flink installation to see if there is any dangling old version jars that you added there. I did a quick dependency check on the Elasticsearch 2 connector, it is correctly pulling in Lucene 5.5.0 only, so this dependency should not pop

Re: Flink Elasticsearch Connector: Lucene Error message

2017-07-17 Thread Fabian Wollert
1.3.0, but i only need the ES 2.X connector working right now, since that's the elasticsearch version we're using. another option would be to upgrade to ES 5 (at elast on dev) to see if its working as well, but that sounds not like fixing the problem for me :-D Cheers Fabian -- *Fabian WollertZ

Re: Kafka Producer - Null Pointer Exception when processing by element

2017-07-17 Thread Tzu-Li (Gordon) Tai
With task chaining as you're saying, could you help clarify how it works  please? Operator can be chained to be executed by a single task thread. See [1] for more details on that. Basically, when two operators are chained together, the output of the first operator is immediately chained to the p