Hi flink. I just ran into the following serialization error in BigPetStore Flink. It appears to be that someone is trying to add elements to a map during serialization. I doubt this is a bug in flink because it would have surely been caught early on. Some possible explanations ... 1 - Probably its a conflict caused by my application's dependencies ? 2 - Or maybe just something about the way the mini-cluster sets up its dependencies?. 3 - Or... Is this related to a guava transitive dependency conflict? DETAILS Our data generator library has these dependencies... Just FYI... [INFO] \- com.github.rnowling.bigpetstore:bigpetstore-data-generator:jar:0.2.1:compile [INFO] +- com.google.guava:guava:jar:18.0:compile [INFO] \- com.google.code.gson:gson:jar:2.3:compile And, heres the exception: Caused by: com.esotericsoftware.kryo.KryoException: java.lang.UnsupportedOperationException Serialization trace: fields (com.github.rnowling.bps.datagenerator.datamodels.Product) products (com.github.rnowling.bps.datagenerator.datamodels.Transaction) at com.esotericsoftware.kryo.serializers.ObjectField.read(ObjectField.java:125) at com.esotericsoftware.kryo.serializers.FieldSerializer.read(FieldSerializer.java:528) at com.esotericsoftware.kryo.Kryo.readClassAndObject(Kryo.java:761) at com.esotericsoftware.kryo.serializers.CollectionSerializer.read(CollectionSerializer.java:116) at com.esotericsoftware.kryo.serializers.CollectionSerializer.read(CollectionSerializer.java:22) at com.esotericsoftware.kryo.Kryo.readObject(Kryo.java:679) at com.esotericsoftware.kryo.serializers.ObjectField.read(ObjectField.java:106) at com.esotericsoftware.kryo.serializers.FieldSerializer.read(FieldSerializer.java:528) at com.esotericsoftware.kryo.Kryo.readClassAndObject(Kryo.java:761) Caused by: java.lang.UnsupportedOperationException at com.google.common.collect.ImmutableMap.put(ImmutableMap.java:326) at com.esotericsoftware.kryo.serializers.MapSerializer.read(MapSerializer.java:144) at com.esotericsoftware.kryo.serializers.MapSerializer.read(MapSerializer.java:21) at com.esotericsoftware.kryo.Kryo.readObject(Kryo.java:679) at com.esotericsoftware.kryo.serializers.ObjectField.read(ObjectField.java:106) ... 18 more |
On Sun, Aug 16, 2015 at 10:38 AM, jay vyas <[hidden email]> wrote:
jay vyas |
Hi! The issue you refer to is: Flink automatically registers types it encounters at Kryo, if the types are not handled by Flink itself. You can turn that off in any program by "ExecutionEnvionment.getConfig().disableAutoTypeRegistration()". I am not sure that causes your problem, as this makes Kryo use tags rather then class names for the types. It may simply be that Kryo does not work with that data type out of the box. It my try to treat the guava immutable map like a regular mp, causing your exception. If that is really the problem, you can register a specific serializer via "ExecuionEnvironment.registerTypeWithKryoSerializer(...)." Let us know what is the problem in the end. Greetings, Stephan On Sun, Aug 16, 2015 at 4:54 PM, jay vyas <[hidden email]> wrote:
|
ah ok gotcha, sort of like this ?
ExecutionEnvironment.getExecutionEnvironment().registerTypeWithKryoSerializer(Product.class, new com.esotericsoftware.kryo.Serializer<Product>() { PS thanks, i think this should work... will update On Sun, Aug 16, 2015 at 11:01 AM, Stephan Ewen <[hidden email]> wrote:
jay vyas |
Hi Jay, this is how you can register a custom Kryo serializer, yes. Flink has this project (https://github.com/magro/kryo-serializers) as a dependency. It contains a lot of Kryo Serializers for common types. They also added support for for Guava's ImmutableMap, but the version we are using (0.27) is too old. Their current release is at 0.36. Can you try adding the higher version to your dependencies? Then, you should be able to add the serializer like this: env.getConfig().registerTypeWithKryoSerializer(ImmutableMap.class, ImmutableMapSerializer.class); On Sun, Aug 16, 2015 at 8:15 PM, jay vyas <[hidden email]> wrote:
|
Free forum by Nabble | Edit this page |