[ 
https://issues.apache.org/jira/browse/FLINK-6321?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16006137#comment-16006137
 ] 

Kostas Kloudas commented on FLINK-6321:
---------------------------------------

Hi [~shashank734], I am trying to reproduce your problem but for that, you 
should share the {{RawSignal}} type and your custom 
{{SignalDeserializationSchema}} (I suppose that this is what you mean by custom 
serializer). 

As you can see from the trace you posted, the problem appears when we are 
trying to deserialize the serializer of your elements that
is encapsulated in the {{NonDuplicatingTypeSerializer}}. So without them it is 
difficult to reproduce the problem. 

As an explanation to why this appears only on RocksDB: that is because this 
statebackend, you serialize and deserialize upon every element, while with FS 
you do not.

Thanks for helping up debug the CEP library.


> RocksDB state backend Checkpointing is not working with KeyedCEP.
> -----------------------------------------------------------------
>
>                 Key: FLINK-6321
>                 URL: https://issues.apache.org/jira/browse/FLINK-6321
>             Project: Flink
>          Issue Type: Bug
>          Components: CEP
>    Affects Versions: 1.2.0
>         Environment: yarn-cluster, RocksDB State backend, Checkpointing every 
> 1000 ms
>            Reporter: Shashank Agarwal
>            Assignee: Kostas Kloudas
>            Priority: Blocker
>             Fix For: 1.3.0
>
>
> Checkpointing is not working with RocksDBStateBackend when using CEP. It's 
> working fine with FsStateBackend and MemoryStateBackend. Application failing 
> every-time.
> {code}
> 04/18/2017 21:53:20   Job execution switched to status FAILING.
> AsynchronousException{java.lang.Exception: Could not materialize checkpoint 
> 46 for operator KeyedCEPPatternOperator -> Map (1/4).}
>       at 
> org.apache.flink.streaming.runtime.tasks.StreamTask$AsyncCheckpointRunnable.run(StreamTask.java:980)
>       at 
> java.util.concurrent.Executors$RunnableAdapter.call(Executors.java:511)
>       at java.util.concurrent.FutureTask.run(FutureTask.java:266)
>       at 
> java.util.concurrent.ThreadPoolExecutor.runWorker(ThreadPoolExecutor.java:1142)
>       at 
> java.util.concurrent.ThreadPoolExecutor$Worker.run(ThreadPoolExecutor.java:617)
>       at java.lang.Thread.run(Thread.java:745)
> Caused by: java.lang.Exception: Could not materialize checkpoint 46 for 
> operator KeyedCEPPatternOperator -> Map (1/4).
>       ... 6 more
> Caused by: java.util.concurrent.CancellationException
>       at java.util.concurrent.FutureTask.report(FutureTask.java:121)
>       at java.util.concurrent.FutureTask.get(FutureTask.java:192)
>       at 
> org.apache.flink.util.FutureUtil.runIfNotDoneAndGet(FutureUtil.java:40)
>       at 
> org.apache.flink.streaming.runtime.tasks.StreamTask$AsyncCheckpointRunnable.run(StreamTask.java:915)
>       ... 5 more
> {code}



--
This message was sent by Atlassian JIRA
(v6.3.15#6346)

Reply via email to