[
https://issues.apache.org/jira/browse/FLINK-6883?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16335823#comment-16335823
]
Shashank Agarwal commented on FLINK-6883:
-----------------------------------------
I am facing something similar issue in 1.4 where i have created save point and
restored both in same 1.4 without changing application code. I am using CEP.
{code}
java.lang.IllegalStateException: Could not initialize keyed state backend.
at
org.apache.flink.streaming.api.operators.AbstractStreamOperator.initKeyedState(AbstractStreamOperator.java:293)
at
org.apache.flink.streaming.api.operators.AbstractStreamOperator.initializeState(AbstractStreamOperator.java:225)
at
org.apache.flink.streaming.runtime.tasks.StreamTask.initializeOperators(StreamTask.java:692)
at
org.apache.flink.streaming.runtime.tasks.StreamTask.initializeState(StreamTask.java:679)
at
org.apache.flink.streaming.runtime.tasks.StreamTask.invoke(StreamTask.java:253)
at org.apache.flink.runtime.taskmanager.Task.run(Task.java:718)
at java.lang.Thread.run(Thread.java:745)
Caused by: java.io.InvalidClassException:
org.apache.flink.cep.scala.pattern.Pattern$$anon$3; invalid descriptor for
field
at java.io.ObjectStreamClass.readNonProxy(ObjectStreamClass.java:723)
at
java.io.ObjectInputStream.readClassDescriptor(ObjectInputStream.java:833)
at
java.io.ObjectInputStream.readNonProxyDesc(ObjectInputStream.java:1609)
at java.io.ObjectInputStream.readClassDesc(ObjectInputStream.java:1521)
at
java.io.ObjectInputStream.readOrdinaryObject(ObjectInputStream.java:1781)
at java.io.ObjectInputStream.readObject0(ObjectInputStream.java:1353)
at java.io.ObjectInputStream.readObject(ObjectInputStream.java:373)
at
org.apache.flink.cep.nfa.NFA$NFASerializer.deserializeCondition(NFA.java:1171)
at
org.apache.flink.cep.nfa.NFA$NFASerializer.deserializeStates(NFA.java:1129)
at org.apache.flink.cep.nfa.NFA$NFASerializer.deserialize(NFA.java:917)
at org.apache.flink.cep.nfa.NFA$NFASerializer.deserialize(NFA.java:820)
at
org.apache.flink.runtime.state.heap.StateTableByKeyGroupReaders$StateTableByKeyGroupReaderV2V3.readMappingsInKeyGroup(StateTableByKeyGroupReaders.java:133)
at
org.apache.flink.runtime.state.heap.HeapKeyedStateBackend.restorePartitionedState(HeapKeyedStateBackend.java:575)
at
org.apache.flink.runtime.state.heap.HeapKeyedStateBackend.restore(HeapKeyedStateBackend.java:446)
at
org.apache.flink.streaming.runtime.tasks.StreamTask.createKeyedStateBackend(StreamTask.java:773)
at
org.apache.flink.streaming.api.operators.AbstractStreamOperator.initKeyedState(AbstractStreamOperator.java:283)
... 6 more
Caused by: java.lang.IllegalArgumentException: illegal signature
at java.io.ObjectStreamField.<init>(ObjectStreamField.java:122)
at java.io.ObjectStreamClass.readNonProxy(ObjectStreamClass.java:721)
... 21 more
{code}
> Serializer for collection of Scala case classes are generated with different
> anonymous class names in 1.3
> ---------------------------------------------------------------------------------------------------------
>
> Key: FLINK-6883
> URL: https://issues.apache.org/jira/browse/FLINK-6883
> Project: Flink
> Issue Type: Bug
> Components: Scala API, Type Serialization System
> Affects Versions: 1.3.0
> Reporter: Tzu-Li (Gordon) Tai
> Assignee: Tzu-Li (Gordon) Tai
> Priority: Blocker
> Labels: flink-rel-1.3.1-blockers
> Fix For: 1.3.1, 1.4.0
>
>
> In the Scala API, serializers are generated using Scala macros (via the
> {{org.apache.flink.streaming.api.scala.createTypeInformation(..)}} util).
> The generated serializers are inner anonymous classes, therefore classnames
> will differ depending on when / order that the serializers are generated.
> From 1.1 / 1.2 to Flink 1.3, the generated classnames for a serializer for a
> collections of case classes (e.g. {{List[SomeUserCaseClass]}}) will be
> different. In other words, the exact same user code written in the Scala API,
> compiling it with 1.1 / 1.2 and with 1.3 will result in different classnames.
> This is problematic for restoring older savepoints that have Scala case class
> collections in their state, because the old serializer cannot be recovered
> (due to the generated classname change).
> For now, I've managed to identify that the root cause for this is that in 1.3
> the {{TypeSerializer}} base class additionally extends the
> {{TypeDeserializer}} interface. Removing this extending resolves the problem.
> The actual reason for why this affects the generated classname is still being
> investigated.
--
This message was sent by Atlassian JIRA
(v7.6.3#76005)