Re-keying / sub-keying a stream without repartitioning

2017-04-21 Thread Elias Levy
This is something that has come up before on the list, but in a different context. I have a need to rekey a stream but would prefer the stream to not be repartitioned. There is no gain to repartitioning, as the new partition key is a composite of the stream key, going from a key of A to a key of

Re: Why TimerService interface in ProcessFunction doesn't have deleteEventTimeTimer

2017-04-21 Thread Ted Yu
Benjamin has an implementation for Hierarchical Timing Wheels (Apache License) : https://github.com/ben-manes/caffeine/blob/master/caffeine/src/main/java/com/github/benmanes/caffeine/cache/TimerWheel.java If there is some interest, we can port the above over. Cheers On Fri, Apr 21, 2017 at

Re: UnilateralSortMerger error (again)

2017-04-21 Thread Flavio Pompermaier
Thanks for the explanation . Is there a way to force this behaviour in a local environment (to try to debug the problem)? On 21 Apr 2017 21:49, "Fabian Hueske" wrote: > Hi Flavio, > > these files are used for spilling data to disk. In your case sorted runs > of records. >

Re: UnilateralSortMerger error (again)

2017-04-21 Thread Fabian Hueske
Hi Flavio, these files are used for spilling data to disk. In your case sorted runs of records. Later all (up to a fanout threshold) these sorted runs are read and merged to get a completely sorted record stream. 2017-04-21 14:09 GMT+02:00 Flavio Pompermaier : > The error

Re: Why TimerService interface in ProcessFunction doesn't have deleteEventTimeTimer

2017-04-21 Thread Gyula Fóra
The timer will actually fire and will be removed at the original time, but we don't trigger any action on it. We also remove the tombstone state afterwards. So we use more memory yes depending on the length and number of timers that were deleted. But it is eventually cleaned up. Gyula Ted Yu

Re: Why TimerService interface in ProcessFunction doesn't have deleteEventTimeTimer

2017-04-21 Thread Ted Yu
A bit curious: wouldn't using "tombstone" markers constitute some memory leak (since Timers are not released) ? Cheers On Fri, Apr 21, 2017 at 12:23 PM, Gyula Fóra wrote: > Hi! > > I thought I would drop my opinion here maybe it is relevant. > > We have used the Flink

Re: Why TimerService interface in ProcessFunction doesn't have deleteEventTimeTimer

2017-04-21 Thread Gyula Fóra
Hi! I thought I would drop my opinion here maybe it is relevant. We have used the Flink internal timer implementation in many of our production applications, this supports the Timer deletion but the deletion actually turned out to be a huge performance bottleneck because of the bad deletion

Re: Generate Timestamps and emit Watermarks - unordered events - Kafka source

2017-04-21 Thread Kostas Kloudas
Hi Luis and Aljoscha, In Flink-1.2 late events were not dropped, but they were processed as normal ones. This is fixed for Flink-1.3 with https://issues.apache.org/jira/browse/FLINK-6205 . I would recommend you to switch to the master branch

R: WELCOME to user@flink.apache.org

2017-04-21 Thread giacom...@libero.it
Dear Users and Apache Flink devs, For each one of my distributed computation, I'm generating and reading the json files produced by the getExecutionPlan() in order to motivate my benchmarks. Is there some guide providing an explaination of the exact meaning of the fields of the

Re: Failed checkpointing on HDFS : Flink don't use the right authentication

2017-04-21 Thread Aljoscha Krettek
+Gordon Could you please have a look at this? You probably know Kafka best by now and have also worked on security related stuff for a while now. I’m afraid I’m not much help here but I’m hoping Gordon can help. Best, Aljoscha > On 21. Apr 2017, at 12:46, Bruno Michelin Rakotondranaivo >

Re: Setting operator parallelism of a running job - Flink 1.2

2017-04-21 Thread Aljoscha Krettek
Hi, both savepoints and checkpoints use the configured state backend. Right now, the only difference between a checkpoint and a savepoint is that the savepoint has additional meta data stored with it that makes it persistent and relocatable. In the future, the (on-disk) format of savepoints and

Re: Setting operator parallelism of a running job - Flink 1.2

2017-04-21 Thread Dominik Safaric
Hi Aljoscha, In other words, jobs must be restarted manually? What about using maxParallelism() at the client level? I would expect that it is complementary to parallelism.default in terms of allowing Flink to handle the parallelism of operators, and changing it in accordance to runtime

Re: Setting operator parallelism of a running job - Flink 1.2

2017-04-21 Thread Aljoscha Krettek
Hi, changing the parallelism is not possible while a job is running (currently). What you would have to do to change the parallelism is create a savepoint and then restore from that savepoint with a different parallelism. This is the savepoints documentation:

Setting operator parallelism of a running job - Flink 1.2

2017-04-21 Thread Dominik Safaric
Hi all, Is it possible to set the operator parallelism using Flink CLI while a job is running? I have a cluster of 4 worker nodes, where each node has 4 CPUs, hence the number of task slots is set to 4, whereas the paralellism.default to 16. However, if a worker fails, whereas the jobs were

Re: Generate Timestamps and emit Watermarks - unordered events - Kafka source

2017-04-21 Thread Aljoscha Krettek
+Kostas and +Dawid Could you please have a look? You two have worked in these parts most recently. I recall that there were some problems when it comes to event time and out-of-order processing in CEP in Flink 1.2 Best, Aljoscha > On 19. Apr 2017, at 15:28, Luis Lázaro

Re: Why TimerService interface in ProcessFunction doesn't have deleteEventTimeTimer

2017-04-21 Thread Aljoscha Krettek
Hi, the reasoning behind the limited user facing API was that we were (are) not sure whether we would be able to support efficient deletion of timers for different ways of storing timers. @Stephan, If I remember correctly you were the strongest advocate for not allowing timer deletion. What’s

Re: Flink slots, threads, task, etc

2017-04-21 Thread Aljoscha Krettek
Hi, there are currently no built-in metrics for InputSplit consumption but I do see that this could be quite helpful. I think you can have a custom RichInputFormat that uses metrics to record stuff, though. I think adding built-in metrics should be possible at this point in the code:

Re: UnilateralSortMerger error (again)

2017-04-21 Thread Flavio Pompermaier
The error appears as soon as some taskmanager generates some inputchannel file. What are those files used for? On Fri, Apr 21, 2017 at 11:53 AM, Flavio Pompermaier wrote: > In another run of the job I had another Exception. Could it be helpful? > > Error obtaining the

Failed checkpointing on HDFS : Flink don't use the right authentication

2017-04-21 Thread Bruno Michelin Rakotondranaivo
Hi, With flink-1.2.0, I want to consume datas from secured kafka 0.10 with SASL_PLAINTEXT protocol using login/pwd from a JAAS file and store them on HDFS in a kerberized cluster with user 'hive' as kerberos principal login. Checkpointing is enabled and states are back end on HDFS

Re: UnilateralSortMerger error (again)

2017-04-21 Thread Flavio Pompermaier
In another run of the job I had another Exception. Could it be helpful? Error obtaining the sorted input: Thread 'SortMerger Reading Thread' terminated due to an exception: Serializer consumed more bytes than the record had. This indicates broken serialization. If you are using custom

RE: Kafka offset commits

2017-04-21 Thread Gwenhael Pasquiers
We need more tests but we think we found the cause for the loss of our kafka consumer offset in kafka 0.10. It might be because of the server-side parameter “offsets.topic.retention.minutes” that defaults to 1440 minutes (1 day). And our flink consumer was “off” for more than a day before

Re: UnilateralSortMerger error (again)

2017-04-21 Thread Flavio Pompermaier
The types I read are: [String, String, String, String, String, String, String, String, String, Boolean, Long, Long, Long, Integer, Integer, Long, String, String, Long, Long, String, Long, String, String, String, String, String, String, String, String, String, String, String, String, String,

Re: UnilateralSortMerger error (again)

2017-04-21 Thread Stephan Ewen
In the past, these errors were most often caused by bugs in the serializers, not in the sorter. What types are you using at that point? The Stack Trace reveals ROW and StringValue, any other involved types? On Fri, Apr 21, 2017 at 9:36 AM, Flavio Pompermaier wrote: > As

Please participate in a research survey on graphs

2017-04-21 Thread Siddhartha Sahu
Hi, My name is Siddhartha Sahu and I am a Master's student at University of Waterloo working on graph processing with Prof. Semih Salihoglu. As part of my research, I am running a survey on how graphs are used in the industry and academia. If you work with any kind of graph technology, such as

Re: Using FlinkML from Java?

2017-04-21 Thread Till Rohrmann
Hi Steve, unfortunately, FlinkML's pipeline mechanism depends on Scala's implicit value feature. Therefore, FlinkML can only be used with Scala if you don't want to construct the pipelines manually (which I wouldn't recommend). Cheers, Till On Thu, Apr 20, 2017 at 6:56 PM, Steve Jerman

Re: Flink memory usage

2017-04-21 Thread Till Rohrmann
Hi Billy, if it's possible that you can share some parts of your code privately with me, then I can try to figure out what's going wrong. Cheers, Till On Thu, Apr 20, 2017 at 6:00 PM, Newport, Billy wrote: > Ok > > The concensus seems to be that it’s us not Flink J So

Re: UnilateralSortMerger error (again)

2017-04-21 Thread Flavio Pompermaier
As suggested by Fabian I set taskmanager.memory.size = 1024 (to force spilling to disk) and the job failed almost immediately.. On Fri, Apr 21, 2017 at 12:33 AM, Flavio Pompermaier wrote: > I debugged a bit the process repeating the job on a sub-slice of the > entire data