You are viewing a plain text version of this content. The canonical link for it is here.
Posted to issues@flink.apache.org by "Timo Walther (JIRA)" <ji...@apache.org> on 2018/11/22 15:23:00 UTC

[jira] [Closed] (FLINK-10674) Fix handling of retractions after clean up

     [ https://issues.apache.org/jira/browse/FLINK-10674?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ]

Timo Walther closed FLINK-10674.
--------------------------------
       Resolution: Fixed
    Fix Version/s: 1.7.0
                   1.6.3
                   1.5.6

Fixed in master: 1878e1168267755356bb332df326625e3049a6c8, 7e86a80b714dac6dc0ae8fda0155d655f623d31e, b480971a1b5d70b9de692f38b69b5578b8ad2355
Fixed in 1.7.0: 3cb24911b9a3fcc081493df735ed1b02a15de3f9, 7e86a80b714dac6dc0ae8fda0155d655f623d31e, b480971a1b5d70b9de692f38b69b5578b8ad2355
Fixed in 1.6.3: d01dbe3a2adb042957355f4d80a77126412fd52d
Fixed in 1.5.6: 5a533e227254d0142181cb1b231da4a9c3d09dfe

> Fix handling of retractions after clean up
> ------------------------------------------
>
>                 Key: FLINK-10674
>                 URL: https://issues.apache.org/jira/browse/FLINK-10674
>             Project: Flink
>          Issue Type: Bug
>          Components: Table API &amp; SQL
>    Affects Versions: 1.6.1
>         Environment: Flink 1.6.0
>            Reporter: ambition
>            Assignee: Timo Walther
>            Priority: Minor
>              Labels: pull-request-available
>             Fix For: 1.5.6, 1.6.3, 1.7.0
>
>         Attachments: image-2018-10-25-14-46-03-373.png
>
>
> Our online Flink Job run about a week,job contain sql :
> {code:java}
> select  `time`,  
>         lower(trim(os_type)) as os_type, 
>         count(distinct feed_id) as feed_total_view  
> from  my_table 
> group by `time`, lower(trim(os_type)){code}
>  
>   then occur NPE: 
>  
> {code:java}
> java.lang.NullPointerException
> at scala.Predef$.Long2long(Predef.scala:363)
> at org.apache.flink.table.functions.aggfunctions.DistinctAccumulator.remove(DistinctAccumulator.scala:109)
> at NonWindowedAggregationHelper$894.retract(Unknown Source)
> at org.apache.flink.table.runtime.aggregate.GroupAggProcessFunction.processElement(GroupAggProcessFunction.scala:124)
> at org.apache.flink.table.runtime.aggregate.GroupAggProcessFunction.processElement(GroupAggProcessFunction.scala:39)
> at org.apache.flink.streaming.api.operators.LegacyKeyedProcessOperator.processElement(LegacyKeyedProcessOperator.java:88)
> at org.apache.flink.streaming.runtime.io.StreamInputProcessor.processInput(StreamInputProcessor.java:202)
> at org.apache.flink.streaming.runtime.tasks.OneInputStreamTask.run(OneInputStreamTask.java:105)
> at org.apache.flink.streaming.runtime.tasks.StreamTask.invoke(StreamTask.java:300)
> at org.apache.flink.runtime.taskmanager.Task.run(Task.java:711)
> at java.lang.Thread.run(Thread.java:745)
> {code}
>  
>  
> View DistinctAccumulator.remove
>  !image-2018-10-25-14-46-03-373.png!
>  
> this NPE should currentCnt = null lead to, so we simple handle like :
> {code:java}
> def remove(params: Row): Boolean = {
>   if(!distinctValueMap.contains(params)){
>     true
>   }else{
>     val currentCnt = distinctValueMap.get(params)
>     // 
>     if (currentCnt == null || currentCnt == 1) {
>       distinctValueMap.remove(params)
>       true
>     } else {
>       var value = currentCnt - 1L
>       if(value < 0){
>         value = 1
>       }
>       distinctValueMap.put(params, value)
>       false
>     }
>   }
> }{code}
>  
> Update:
> Because state clean up happens in processing time, it might be
>  the case that retractions are arriving after the state has
>  been cleaned up. Before these changes, a new accumulator was
>  created and invalid retraction messages were emitted. This
>  change drops retraction messages for which no accumulator
>  exists. 



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