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

2018-01-08 Thread Kyle Hamlin
+Aljoscha Krettek I setup my project using the template you suggested and I'm able to bucket and write files locally. I also want to test writing to s3 but I don't know how to configure the `sbt run` command to tell the FlinkMiniCluster to use the flink-s3-fs-hadoop-1.4.0.jar and a flink-conf.yaml

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} were no longer in

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

2018-01-05 Thread Kyle Hamlin
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} were no longer in Flink and I use them in my custom bucketer and to writer to write Avro out to Parquet. I tried adding classlo

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

2018-01-04 Thread Aljoscha Krettek
I think this might be happening because partial Hadoop dependencies are in the user jar and the rest is only available from the Hadoop deps that come bundled with Flink. For example, I noticed that you have Hadoop-common as a dependency which probably ends up in your Jar. > On 4. Jan 2018, at 1

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

2018-01-04 Thread Stephan Ewen
@Kyle: Please also check if you have any Hadoop classes in your user jar. There should be none, Hadoop should only be in the Flink classpath. Fixing the project Maven setup (making sure Hadoop and Flink core dependencies are provided) should work. To do that, you can for example use the latest qu

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

2018-01-04 Thread Stephan Ewen
Hi! This looks indeed like a class-loading 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 way to ma

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"