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

ASF GitHub Bot commented on FLINK-6571:
---------------------------------------

Github user StephanEwen commented on the issue:

    https://github.com/apache/flink/pull/5394
  
    I have often handled it like one of the below variants. What do you think 
about that pattern?
    
    ### Variant 1: Handle interruption if still running
    ```java
    public void run(SourceContext<T> ctx) throws Exception {
        while (running) {
            try {
                // do stuff
                Thread.sleep(20);
            } catch (InterruptedException e) {
                // restore interruption flag
                Thread.currentThread().interrupt();
                if (running) {
                    throw new FlinkException("interrupted while still running", 
e);
                }
                // else fall through the loop
        }
    }
    ```
    
    ### Variant 2: Simple let InterruptedException bubble out
    
    This variant is also fine, because the Task status is set to CANCELED 
before the interruption, so any exception bubbling out be suppresses.
    
    ```java
    public void run(SourceContext<T> ctx) throws Exception {
        while (running) {
            // do stuff
    
            // the InterruptedException from here simply fails the execution
            Thread.sleep(20);
        }
    }
    ```


> InfiniteSource in SourceStreamOperatorTest should deal with 
> InterruptedExceptions
> ---------------------------------------------------------------------------------
>
>                 Key: FLINK-6571
>                 URL: https://issues.apache.org/jira/browse/FLINK-6571
>             Project: Flink
>          Issue Type: Improvement
>          Components: Tests
>    Affects Versions: 1.3.0, 1.4.0, 1.5.0
>            Reporter: Chesnay Schepler
>            Assignee: Chesnay Schepler
>            Priority: Blocker
>              Labels: test-stability
>             Fix For: 1.5.0
>
>
> So this is a new one: i got a failing test 
> ({{testNoMaxWatermarkOnAsyncStop}}) due to an uncatched InterruptedException.
> {code}
> [00:28:15] Tests run: 7, Failures: 0, Errors: 1, Skipped: 0, Time elapsed: 
> 0.828 sec <<< FAILURE! - in 
> org.apache.flink.streaming.runtime.operators.StreamSourceOperatorTest
> [00:28:15] 
> testNoMaxWatermarkOnAsyncStop(org.apache.flink.streaming.runtime.operators.StreamSourceOperatorTest)
>   Time elapsed: 0 sec  <<< ERROR!
> [00:28:15] java.lang.InterruptedException: sleep interrupted
> [00:28:15]    at java.lang.Thread.sleep(Native Method)
> [00:28:15]    at 
> org.apache.flink.streaming.runtime.operators.StreamSourceOperatorTest$InfiniteSource.run(StreamSourceOperatorTest.java:343)
> [00:28:15]    at 
> org.apache.flink.streaming.api.operators.StreamSource.run(StreamSource.java:87)
> [00:28:15]    at 
> org.apache.flink.streaming.runtime.operators.StreamSourceOperatorTest.testNoMaxWatermarkOnAsyncStop(StreamSourceOperatorTest.java:176)
> {code}



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

Reply via email to