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

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

Github user StephanEwen commented on a diff in the pull request:

    https://github.com/apache/flink/pull/750#discussion_r40333050
  
    --- Diff: 
flink-clients/src/main/java/org/apache/flink/client/CliFrontend.java ---
    @@ -534,7 +536,70 @@ public int compare(JobStatusMessage o1, 
JobStatusMessage o2) {
                                                "RunningJobs. Instead the 
response is of type " + result.getClass() + ".");
                        }
                }
    -           catch (Throwable t) {
    +           catch (Exception t) {
    +                   return handleError(t);
    +           }
    +   }
    +
    +   /**
    +    * Executes the STOP action.
    +    * 
    +    * @param args Command line arguments for the stop action.
    +    */
    +   protected int stop(String[] args) {
    +           LOG.info("Running 'stop' command.");
    +
    +           StopOptions options;
    +           try {
    +                   options = CliFrontendParser.parseStopCommand(args);
    +           }
    +           catch (CliArgsException e) {
    +                   return handleArgException(e);
    +           }
    +           catch (Exception t) {
    +                   return handleError(t);
    +           }
    +
    +           // evaluate help flag
    +           if (options.isPrintHelp()) {
    +                   CliFrontendParser.printHelpForStop();
    +                   return 0;
    +           }
    +
    +           String[] stopArgs = options.getArgs();
    +           JobID jobId;
    +
    +           if (stopArgs.length > 0) {
    +                   String jobIdString = stopArgs[0];
    +                   try {
    +                           jobId = new 
JobID(StringUtils.hexStringToByte(jobIdString));
    +                   }
    +                   catch (Exception e) {
    +                           LOG.error("Error: The value for the Job ID is 
not a valid ID.");
    +                           System.out.println("Error: The value for the 
Job ID is not a valid ID.");
    +                           return 1;
    +                   }
    +           }
    +           else {
    +                   LOG.error("Missing JobID in the command line 
arguments.");
    +                   System.out.println("Error: Specify a Job ID to stop a 
job.");
    +                   return 1;
    +           }
    +
    +           try {
    +                   ActorGateway jobManager = getJobManagerGateway(options);
    +                   Future<Object> response = jobManager.ask(new 
StopJob(jobId), askTimeout);
    +
    +                   Object rc = Await.result(response, askTimeout);
    +
    +                   if (rc instanceof StoppingFailure) {
    --- End diff --
    
    I think that we decided not to use the built-in `Failure` object when 
transporting Exceptions between processes, if the Exceptions may be of 
user-defined classes (class loading issues). Using custom message types that 
explicitly require a `SerializedThrowable` makes it safer for users to 
recognize the need to supply a classloader.


> Add "stop" signal to cleanly shutdown streaming jobs
> ----------------------------------------------------
>
>                 Key: FLINK-2111
>                 URL: https://issues.apache.org/jira/browse/FLINK-2111
>             Project: Flink
>          Issue Type: Improvement
>          Components: Distributed Runtime, JobManager, Local Runtime, 
> Streaming, TaskManager, Webfrontend
>            Reporter: Matthias J. Sax
>            Assignee: Matthias J. Sax
>            Priority: Minor
>
> Currently, streaming jobs can only be stopped using "cancel" command, what is 
> a "hard" stop with no clean shutdown.
> The new introduced "stop" signal, will only affect streaming source tasks 
> such that the sources can stop emitting data and shutdown cleanly, resulting 
> in a clean shutdown of the whole streaming job.
> This feature is a pre-requirment for 
> https://issues.apache.org/jira/browse/FLINK-1929



--
This message was sent by Atlassian JIRA
(v6.3.4#6332)

Reply via email to