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

Timo Walther commented on FLINK-11443:
--------------------------------------

[~tivan] Changing SQL queries is not supported so far. If a query is changed, 
the optimizer might chose a different plan such that Flink's internal state can 
not be mapped anymore. This is a complex problem that won't be solved in the 
near future. We recommend to built up the state from scratch (maybe in parallel 
to the old query) and then perform a switch to the new query once the new job 
is up to date.

>  Recovering from save point error after adding "sum (constant)"
> ---------------------------------------------------------------
>
>                 Key: FLINK-11443
>                 URL: https://issues.apache.org/jira/browse/FLINK-11443
>             Project: Flink
>          Issue Type: Bug
>          Components: Table API & SQL
>    Affects Versions: 1.6.2, 1.6.3
>            Reporter: tivanli
>            Priority: Minor
>
> h3. Resuming from savepoint error when I add a "sum (2)" column to flink sql .
>  
> {code:java}
> org.apache.flink.util.StateMigrationException: State migration is currently 
> not supported.
>       at 
> org.apache.flink.util.StateMigrationException.notSupported(StateMigrationException.java:42)
>       at 
> org.apache.flink.runtime.state.RegisteredKeyValueStateBackendMetaInfo.resolveKvStateCompatibility(RegisteredKeyValueStateBackendMetaInfo.java:212)
>       at 
> org.apache.flink.contrib.streaming.state.RocksDBKeyedStateBackend.tryRegisterKvStateInformation(RocksDBKeyedStateBackend.java:1336)
>       at 
> org.apache.flink.contrib.streaming.state.RocksDBKeyedStateBackend.createInternalState(RocksDBKeyedStateBackend.java:1391)
>       at 
> org.apache.flink.runtime.state.KeyedStateFactory.createInternalState(KeyedStateFactory.java:47)
>       at 
> org.apache.flink.runtime.state.ttl.TtlStateFactory.createStateAndWrapWithTtlIfEnabled(TtlStateFactory.java:63)
>       at 
> org.apache.flink.runtime.state.AbstractKeyedStateBackend.getOrCreateKeyedState(AbstractKeyedStateBackend.java:238)
>       at 
> org.apache.flink.streaming.api.operators.AbstractStreamOperator.getOrCreateKeyedState(AbstractStreamOperator.java:562)
>       at 
> org.apache.flink.streaming.runtime.operators.windowing.WindowOperator.open(WindowOperator.java:240)
>       at 
> org.apache.flink.streaming.runtime.tasks.StreamTask.openAllOperators(StreamTask.java:424)
>       at 
> org.apache.flink.streaming.runtime.tasks.StreamTask.invoke(StreamTask.java:290)
>       at org.apache.flink.runtime.taskmanager.Task.run(Task.java:711)
>       at java.lang.Thread.run(Thread.java:748)
> {code}
>  Originally running sql:
> {code:java}
> SELECT TUMBLE_START(rowtime, INTERVAL '1' minute) as dtEventTime,
> word,
> sum(frequency) as frequency
> FROM test
> GROUP BY word,
> TUMBLE(rowtime, INTERVAL '1' minute)
> {code}
>  Modify sql to increase sum function
> {code:java}
> SELECT TUMBLE_START(rowtime, INTERVAL '1' minute) as dtEventTime,
> word,
> sum(frequency) as frequency,
> sum(2) as s2
> FROM test
> GROUP BY word,
> TUMBLE(rowtime, INTERVAL '1' minute)
> {code}
>  
>  



--
This message was sent by Atlassian JIRA
(v7.6.3#76005)

Reply via email to