Re: BucketingSink broken in flink 1.4.0 ?

2018-01-10 Thread Kyle Hamlin
I'm having similar issues after moving from 1.3..2 to 1.4.0 *My mailing list thread: *BucketingSink doesn't work anymore moving from 1.3.2 to 1.4.0 I'm not ac

Re: Testing Flink 1.4 unable to write to s3 locally

2018-01-09 Thread Kyle Hamlin
! > > For the exception in the "local file path" case, we already had a reason: > > - You were having Hadoop code in your user code jar. Either not putting > the Hadoop dependency into your jar (rather in Flink's lib directory) or > setting classloading to &qu

Re: Testing Flink 1.4 unable to write to s3 locally

2018-01-08 Thread Kyle Hamlin
ava:1154) at org.apache.flink.streaming.connectors.fs.bucketing.BucketingSink.initFileSystem(BucketingSink.java:411) at org.apache.flink.streaming.connectors.fs.bucketing.BucketingSink.initializeState(BucketingSink.java:355) ... 9 more Caused by: java.lang.ClassCastException On Mon, Jan 8, 2018 at 9:21 PM Kyle Hamlin wrote: > H

Re: Testing Flink 1.4 unable to write to s3 locally

2018-01-08 Thread Kyle Hamlin
og files with more log entries? > > That would be helpful... > > > On Wed, Jan 3, 2018 at 5:26 PM, Kyle Hamlin wrote: > >> Hello Stephan & Nico, >> >> Here is the full stacktrace, its not much more than what I originally >> posted. I remember s

Re: BucketingSink doesn't work anymore moving from 1.3.2 to 1.4.0

2018-01-08 Thread Kyle Hamlin
'm able to the flink-conf.yaml and flink-s3-fs-hadoop-1.4.0.jar? On Fri, Jan 5, 2018 at 7:50 PM Kyle Hamlin wrote: > Also, I'm not using hdfs I'm trying to sink to s3. > > On Fri, Jan 5, 2018 at 6:18 PM Kyle Hamlin wrote: > >> I have the hadoop-common.jar in my buil

Dynamically get schema from element to pass to AvroParquetWriter

2018-01-05 Thread Kyle Hamlin
I implemented an Avro to Parquet writer which previously took an Avro schema in as a string to the constructor and passed it to the AvroParquetWriter. Now I'm wondering if there is a way to get the schema from the element and pass to the AvroParquetWriter. I tried grabbing the schema from the eleme

Re: BucketingSink doesn't work anymore moving from 1.3.2 to 1.4.0

2018-01-05 Thread Kyle Hamlin
Also, I'm not using hdfs I'm trying to sink to s3. On Fri, Jan 5, 2018 at 6:18 PM Kyle Hamlin wrote: > I have the hadoop-common.jar in my build.sbt because I was having issues > compiling my jar after moving from 1.3.2 to 1.4.0 because > org.apache.hadoop.fs.{FileSystem, Path

Re: BucketingSink doesn't work anymore moving from 1.3.2 to 1.4.0

2018-01-05 Thread Kyle Hamlin
ing issue - it looks like "RpcEngine" > and "ProtobufRpcEngine" are loaded via different classloaders. > > Can you try the following: > > - In your flink-conf.yml, set classloader.resolve-order: parent-first > > If that fixes the issue, then we can look at a w

Re: Can't call getProducedType on Avro messages with array types

2018-01-05 Thread Kyle Hamlin
So I just added the dependency but didn't change the getProducedType method and it worked fine. Would you expect that to be the case? On Fri, Jan 5, 2018 at 5:43 PM Aljoscha Krettek wrote: > Yes, that should do the trick. > > > On 5. Jan 2018, at 18:37, Kyle Hamlin wrote: &

Re: Can't call getProducedType on Avro messages with array types

2018-01-05 Thread Kyle Hamlin
]) } On Thu, Jan 4, 2018 at 11:08 AM Aljoscha Krettek wrote: > Hi, > > I think you might be able to use AvroTypeInfo which you can use by > including the flink-avro dependencies. Is that an option for you? > > Best, > Aljoscha > > > On 3. Jan 2018, at 21:34, Kyle

Can't call getProducedType on Avro messages with array types

2018-01-03 Thread Kyle Hamlin
Hi, It appears that Kryo can't properly extract/deserialize Avro array types. I have a very simple Avro schema that has an array type and when I remove the array field the error is not thrown. Is there any way around this without using a specific type? *Avro Schema:* { "type": "record", "

BucketingSink doesn't work anymore moving from 1.3.2 to 1.4.0

2018-01-03 Thread Kyle Hamlin
Hello, After moving to Flink 1.4.0 I'm getting the following error. I can't find anything online that addresses it. Is it a Hadoop dependency issue? Here are my project dependencies: libraryDependencies ++= Seq( "org.apache.flink" %% "flink-scala" % flinkVersion % Provided, "org.apache.flink"

Re: Testing Flink 1.4 unable to write to s3 locally

2018-01-03 Thread Kyle Hamlin
; >> Nico >> >> [1] >> >> https://ci.apache.org/projects/flink/flink-docs-release-1.4/ops/deployment/aws.html#shaded-hadooppresto-s3-file-systems-recommended >> >> On 03/01/18 10:22, Stephan Ewen wrote: >> > Hi Kyle! >> > >> > Is ther

Re: Separate checkpoint directories

2018-01-03 Thread Kyle Hamlin
heckpoint data for > each operator instance. > > Best, > Stefan > > > Am 31.12.2017 um 17:10 schrieb Kyle Hamlin : > > Flink 1.4 added regex pattern matching for FlinkKafkaConsumer's which is a > neat feature. I would like to use this feature, but I'm won

Re: Separate checkpoint directories

2018-01-03 Thread Kyle Hamlin
ll already create proper subdirectories > and filenames and can identify the right checkpoint data for each operator > instance. > > Best, > Stefan > >> Am 31.12.2017 um 17:10 schrieb Kyle Hamlin : >> >> Flink 1.4 added regex pattern matching for FlinkKafkaCons

Separate checkpoint directories

2017-12-31 Thread Kyle Hamlin
Flink 1.4 added regex pattern matching for FlinkKafkaConsumer's which is a neat feature. I would like to use this feature, but I'm wondering how that impacts the FsStateBackend checkpointing mechanism. Before I would subscribe to one topic and set a checkpoint path specific to that topic for exampl

Testing Flink 1.4 unable to write to s3 locally

2017-12-31 Thread Kyle Hamlin
Hi, When testing Flink 1.4 locally the error below keeps getting thrown. I've followed the setup by moving the flink-s3-fs-presto.jar from the opt/ folder to the lib/ folder. Is there something additional I need to do? java.lang.NoClassDefFoundError: Could not initialize class org.apache.flink.fs