Re: Flink Kafka Producer Exception

2017-12-13 Thread Tzu-Li (Gordon) Tai
Hi Navneeth, The exception you are getting is a Kafka NetworkException. From the provided information I can’t really tell much and can only guess, but are you sure that the client / broker versions match? It seems like that you are using 0.10; the default client version in the Flink Kafka 0.10

Re: how does time-windowed join and Over-Window Aggregation implemented in flink SQL?

2017-12-13 Thread Xingcan Cui
Hi Yan Zhou, as you may have noticed, the SQL level stream join was not built on top of some join APIs but was implemented with the low-level CoProcessFunction (see TimeBoundedStreamInnerJoin.scala

Re: Flink flick cancel vs stop

2017-12-13 Thread Elias Levy
I am re-upping this thread now that FlinkKafkaProducer011 is out. The new producer, when used with the exactly once semantics, has the rather troublesome behavior that it will fallback to at-most-once, rather than at-least-once, if the job is down for longer than the Kafka broker's

FlinkKafkaProducer011 and Flink 1.4.0 Kafka docs

2017-12-13 Thread Elias Levy
Looks like the Flink Kafka connector page, in the Producer section , is missing a section for the new FlinkKafkaProducer011 producer. Given that the new producer no longer has a static

how does time-windowed join and Over-Window Aggregation implemented in flink SQL?

2017-12-13 Thread Yan Zhou [FDS Science]
Hi, I am building a data pipeline with a lot of streaming join and over window aggregation. And flink SQL have these feature supported. However, there is no similar DataStream APIs provided(maybe there is and I didn't find them. please point out if there is). I got confused because I assume

Re: Flink Kafka Producer Exception

2017-12-13 Thread Navneeth Krishnan
Hi, I'm receiving this error and due to which I'm not able to run my job. Any help is greatly appreciated. Thanks. On Tue, Dec 12, 2017 at 10:21 AM, Navneeth Krishnan < reachnavnee...@gmail.com> wrote: > Hi, > > I have a kafka source and sink in my pipeline and when I start my job I > get this

Re: Custom Metrics

2017-12-13 Thread Navneeth Krishnan
Thanks Pitor. I have couple more questions related to metrics. I use Influx db reporter to report flink metrics and I see a lot of metrics are bring reported. Is there a way to select only a subset of metrics that we need to monitor the application? Also, Is there a way to specify custom metics

Re: Watermark in broadcast

2017-12-13 Thread Seth Wiesman
Quick follow up question. Is there some way to notify a TimestampAssigner that is consuming from an idle source? [cid:image001.png@01D3740B.CADE87C0] Seth Wiesman | Software Engineer, Data
 4 World Trade Center, 46th Floor, New York, NY 10007
 From: Seth Wiesman

Re: Watermark in broadcast

2017-12-13 Thread Seth Wiesman
Hi Timo, I think you are correct. This stream is consumed from Kafka and the number of partitions is much less than the parallelism of the program so there would be many partitions that never forward watermarks greater than Long.Min_Value. Thank you for the quick response.

Re: Watermark in broadcast

2017-12-13 Thread Timo Walther
Hi Seth, are you sure that all partitions of the broadcasted stream send a watermark? processWatermark is only called if a minimum watermark arrived from all partitions. Regards, Timo Am 12/13/17 um 5:10 PM schrieb Seth Wiesman: Hi, How are watermarks propagated during a broadcast

Watermark in broadcast

2017-12-13 Thread Seth Wiesman
Hi, How are watermarks propagated during a broadcast partition? I have a TwoInputStreamTransformation that takes a broadcast stream as one of its inputs. Both streams are assigned timestamps and watermarks before being connected however I only ever see watermarks from my non-broadcast stream.

Re: ClassCastException when using RowTypeInfo

2017-12-13 Thread Timo Walther
Hi Madan, this is definitely a bug. The Row type has mostly been added for the Table & SQL API and has not tested for expression keys. But in general I would use a tuple in your case as they are more efficient. The `registerType` is only necessary for generic types serialized with Kryo. I

ClassCastException when using RowTypeInfo

2017-12-13 Thread madan
Hi, Below is sample code I am trying with, StreamExecutionEnvironment env = StreamExecutionEnvironment. getExecutionEnvironment(); TypeInformation[] types = new TypeInformation[] {BasicTypeInfo.INT_TYPE_INFO, BasicTypeInfo.STRING_TYPE_INFO, BasicTypeInfo.DOUBLE_TYPE_INFO,

Re: [ANNOUNCE] Apache Flink 1.4.0 released

2017-12-13 Thread Vishal Santoshi
+1 On Wed, Dec 13, 2017 at 8:13 AM, Till Rohrmann wrote: > Thanks a lot Aljoscha for being the release manager and to the whole Flink > community for this release :-) > > @Vishal: flink-cep/flink-connector were deployed to maven central. It just > might take a while

Re: when does the timed window ends?

2017-12-13 Thread Aljoscha Krettek
Hi, Yes, those last two comments about the watermark and window triggering are correct. The watermark either has to advance based on events or based on some continuous generation. Best, Aljoscha > On 13. Dec 2017, at 06:08, Jinhua Luo wrote: > > Unless I generate

Re: [ANNOUNCE] Apache Flink 1.4.0 released

2017-12-13 Thread Till Rohrmann
Thanks a lot Aljoscha for being the release manager and to the whole Flink community for this release :-) @Vishal: flink-cep/flink-connector were deployed to maven central. It just might take a while until they show up I guess. Cheers, Till On Wed, Dec 13, 2017 at 4:23 AM, Vishal Santoshi

Re: netty conflict using lettuce redis client

2017-12-13 Thread Timo Walther
Hi, we just released Flink 1.4.0 [1]. Maybe it is possible for you to upgrade? One of the greatest features is improved classloading and better dependency management. I think this would be the easiest solution for you. Otherwise let us know if you still need help. Regards, Timo [1]

Re: [ANNOUNCE] Apache Flink 1.4.0 released

2017-12-13 Thread Aljoscha Krettek
Hi, @Flavio, I think the update should be frictionless. At the end of the release post are some notes and only if you would be affected by this do you have to change something. @Hao Sun Are you using that on EMR? I think in that case you don't have to include anything. If not, it might be

netty conflict using lettuce redis client

2017-12-13 Thread Jinhua Luo
Hi All, The io.netty package included in flnk 1.3.2 is 4.0.23, while the latest lettuce-core (4.4) depends on netty 4.0.35. If I include netty 4.0.35 in the app jar, it would throw java.nio.channels.UnresolvedAddressException. It seems the netty classes are mixed between versions from app jar

Re: Kafka topic partition skewness causes watermark not being emitted

2017-12-13 Thread Gerard Garcia
Thanks Gordon. Don't worry, I'll be careful to not have empty partitions until the next release. Also, I'll keep an eye to FLINK-5479 and if at some point I see that there is a fix and the issue bothers us too much I'll try to apply the patch myself to the latest stable release. Gerard On Wed,

Re: Cannot load user classes

2017-12-13 Thread Chesnay Schepler
Can you show us the dependency section of your pom? On 13.12.2017 08:25, Soheil Pourbafrani wrote: Hey, I wrote a code using Flink and creating fat jar using maven, I can errorlessly run it on a remote cluster. Trying to run it without creating a fat jar and directly from IDE I got the error

Re: Kafka topic partition skewness causes watermark not being emitted

2017-12-13 Thread Tzu-Li (Gordon) Tai
Hi, I've just elevated FLINK-5479 to BLOCKER for 1.5. Unfortunately, AFAIK there is no easy workaround solution for this issue yet in the releases so far. The min watermark logic that controls per-partition watermark emission is hidden inside the consumer, making it hard to work around it. One

Re: Off heap memory issue

2017-12-13 Thread Piotr Nowojski
Hi, OOMs from metaspace probably mean that your jars are not releasing some resources: https://ci.apache.org/projects/flink/flink-docs-release-1.3/monitoring/debugging_classloading.html#unloading-of-dynamically-loaded-classes