Skip to content

Commit

Permalink
[FLINK-6787] Fix Job-/StoppableException extend FlinkException
Browse files Browse the repository at this point in the history
  • Loading branch information
FangYongs authored and aljoscha committed Aug 31, 2017
1 parent e6fddbc commit ddf62b1
Show file tree
Hide file tree
Showing 9 changed files with 28 additions and 9 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -382,7 +382,7 @@ else if (prog.isUsingInteractiveMode()) {
// invoke main method
prog.invokeInteractiveModeForExecution();
if (lastJobExecutionResult == null && factory.getLastEnvCreated() == null) {
throw new ProgramMissingJobException();
throw new ProgramMissingJobException("The program didn't contain a Flink job.");
}
if (isDetached()) {
// in detached mode, we execute the whole user code to extract the Flink job, afterwards we run it here
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -18,12 +18,18 @@

package org.apache.flink.client.program;

import org.apache.flink.util.FlinkException;

/**
* Exception used to indicate that no job was executed during the invocation of a Flink program.
*/
public class ProgramMissingJobException extends Exception {
public class ProgramMissingJobException extends FlinkException {
/**
* Serial version UID for serialization interoperability.
*/
private static final long serialVersionUID = -1964276369605091101L;

public ProgramMissingJobException(String message) {
super(message);
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -18,10 +18,12 @@

package org.apache.flink.runtime;

import org.apache.flink.util.FlinkException;

/**
* Indicates that a job has failed.
*/
public class JobException extends Exception {
public class JobException extends FlinkException {

private static final long serialVersionUID = 1275864691743020176L;

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -18,10 +18,12 @@

package org.apache.flink.runtime;

import org.apache.flink.util.FlinkException;

/**
* Indicates that a job is not stoppable.
*/
public class StoppingException extends Exception {
public class StoppingException extends FlinkException {

private static final long serialVersionUID = -721315728140810694L;

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -18,11 +18,13 @@

package org.apache.flink.runtime.client;

import org.apache.flink.util.FlinkException;

/**
* Exception which is thrown when the {@link JobClientActor} wants to submit a job to
* the job manager but has not found one after a given timeout interval.
*/
public class JobClientActorConnectionTimeoutException extends Exception {
public class JobClientActorConnectionTimeoutException extends FlinkException {
private static final long serialVersionUID = 2287747430528388637L;

public JobClientActorConnectionTimeoutException(String msg) {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -18,11 +18,13 @@

package org.apache.flink.runtime.client;

import org.apache.flink.util.FlinkException;

/**
* Exception which is thrown by the {@link JobClientActor} if it has not heard back from the job
* manager after it has attempted to register for a job within a given timeout interval.
*/
public class JobClientActorRegistrationTimeoutException extends Exception {
public class JobClientActorRegistrationTimeoutException extends FlinkException {
private static final long serialVersionUID = 8762463142030454853L;

public JobClientActorRegistrationTimeoutException(String msg) {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -18,11 +18,13 @@

package org.apache.flink.runtime.client;

import org.apache.flink.util.FlinkException;

/**
* Exception which is thrown by the {@link JobClientActor} if it has not heard back from the job
* manager after it has submitted a job to it within a given timeout interval.
*/
public class JobClientActorSubmissionTimeoutException extends Exception {
public class JobClientActorSubmissionTimeoutException extends FlinkException {
private static final long serialVersionUID = 8762463142030454853L;

public JobClientActorSubmissionTimeoutException(String msg) {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -19,13 +19,14 @@
package org.apache.flink.runtime.client;

import org.apache.flink.api.common.JobID;
import org.apache.flink.util.FlinkException;

/**
* This exception is the base exception for all exceptions that denote any failure during
* the execution of a job. The JobExecutionException and its subclasses are thrown by
* the {@link JobClient}.
*/
public class JobExecutionException extends Exception {
public class JobExecutionException extends FlinkException {

private static final long serialVersionUID = 2818087325120827525L;

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -18,10 +18,12 @@

package org.apache.flink.runtime.jobmaster;

import org.apache.flink.util.FlinkException;

/**
* Base exception thrown by the {@link JobMaster}.
*/
public class JobManagerException extends Exception {
public class JobManagerException extends FlinkException {

private static final long serialVersionUID = -7290962952242188064L;

Expand Down

0 comments on commit ddf62b1

Please sign in to comment.