Skip to content
Open
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
Original file line number Diff line number Diff line change
Expand Up @@ -46,6 +46,7 @@
import org.apache.beam.sdk.values.PCollectionTuple;
import org.apache.beam.sdk.values.TupleTag;
import org.apache.beam.sdk.values.TupleTagList;
import org.apache.beam.vendor.guava.v32_1_2_jre.com.google.common.base.Throwables;
import org.checkerframework.checker.nullness.qual.MonotonicNonNull;
import org.joda.time.Duration;

Expand Down Expand Up @@ -613,12 +614,32 @@ private void executeAsync(Callable<Void> callable) throws UserCodeExecutionExcep
private static <T> void parseAndThrow(Future<T> future, ExecutionException e)
throws UserCodeExecutionException {
future.cancel(true);
if (e.getCause() == null) {
throw new UserCodeExecutionException(e);

try {
UserCodeExecutionException genericException = null;
for (Throwable throwable : Throwables.getCausalChain(e)) {
if (throwable instanceof UserCodeQuotaException) {
throw (UserCodeQuotaException) throwable;
} else if (throwable instanceof UserCodeTimeoutException) {
throw (UserCodeTimeoutException) throwable;
} else if (throwable instanceof UserCodeRemoteSystemException) {
throw (UserCodeRemoteSystemException) throwable;
} else if (genericException == null && throwable instanceof UserCodeExecutionException) {
genericException = (UserCodeExecutionException) throwable;
}
}
if (genericException != null) {
throw genericException;
}
} catch (IllegalArgumentException iae) {
// Circular reference detected in causal chain
throw new UserCodeExecutionException(
"circular reference detected in exception causal chain", e);
}
Throwable cause = checkStateNotNull(e.getCause());
if (cause instanceof UserCodeQuotaException) {
throw new UserCodeQuotaException(cause);

Throwable cause = e.getCause();
if (cause == null) {
throw new UserCodeExecutionException(e);
}
throw new UserCodeExecutionException(cause);
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -104,6 +104,34 @@ public void givenCallerThrowsUserCodeExecutionException_emitsIntoFailurePCollect
pipeline.run();
}

@Test
public void givenCallerThrowsNonUserCodeException_emitsWrappedUserCodeExecutionException() {
Result<Response> result =
pipeline
.apply(Create.of(new Request("a")))
.apply(Call.of(new CallerThrowsRuntimeException(), NON_DETERMINISTIC_RESPONSE_CODER));

PCollection<ApiIOError> failures = result.getFailures();
PAssert.thatSingleton(countStackTracesOf(failures, UserCodeExecutionException.class))
.isEqualTo(1L);

pipeline.run();
}

@Test
public void givenCallerThrowsCircularCausalChain_emitsUserCodeExecutionException() {
Result<Response> result =
pipeline
.apply(Create.of(new Request("a")))
.apply(Call.of(new CallerThrowsCircularCause(), NON_DETERMINISTIC_RESPONSE_CODER));

PCollection<ApiIOError> failures = result.getFailures();
PAssert.thatSingleton(countStackTracesOf(failures, UserCodeExecutionException.class))
.isEqualTo(1L);

pipeline.run();
}

@Test
public void givenCallerThrowsQuotaException_emitsIntoFailurePCollection() {
Result<Response> result =
Expand Down Expand Up @@ -142,22 +170,169 @@ public void givenCallerTimeout_emitsFailurePCollection() {
}

@Test
public void givenCallerThrowsTimeoutException_emitsFailurePCollection() {
public void givenCallerThrowsTimeoutException_thenPreservesExceptionType() {
Result<Response> result =
pipeline
.apply(Create.of(new Request("a")))
.apply(Call.of(new CallerThrowsTimeout(), NON_DETERMINISTIC_RESPONSE_CODER));

PCollection<ApiIOError> failures = result.getFailures();
PAssert.thatSingleton(countStackTracesOf(failures, UserCodeExecutionException.class))
.isEqualTo(1L);
.isEqualTo(0L);
PAssert.thatSingleton(countStackTracesOf(failures, UserCodeQuotaException.class)).isEqualTo(0L);
PAssert.thatSingleton(countStackTracesOf(failures, UserCodeTimeoutException.class))
.isEqualTo(1L);

pipeline.run();
}

@Test
public void givenCallerThrowsRemoteSystemException_thenPreservesExceptionType() {
Result<Response> result =
pipeline
.apply(Create.of(new Request("a")))
.apply(
Call.of(new CallerThrowsRemoteSystemException(), NON_DETERMINISTIC_RESPONSE_CODER));

PCollection<ApiIOError> failures = result.getFailures();
PAssert.thatSingleton(countStackTracesOf(failures, UserCodeRemoteSystemException.class))
.isEqualTo(1L);
PAssert.thatSingleton(countStackTracesOf(failures, UserCodeExecutionException.class))
.isEqualTo(0L);

pipeline.run();
}

@Test
public void givenNestedExecutionException_thenPreservesExceptionType() {
Result<Response> result =
pipeline
.apply(Create.of(new Request("a")))
.apply(
Call.of(
new CallerThrowsNestedExecutionException(), NON_DETERMINISTIC_RESPONSE_CODER));

PCollection<ApiIOError> failures = result.getFailures();
PAssert.thatSingleton(countStackTracesOf(failures, UserCodeExecutionException.class))
.isEqualTo(1L);
PAssert.thatSingleton(countStackTracesOf(failures, UserCodeTimeoutException.class))
.isEqualTo(0L);
PAssert.thatSingleton(countStackTracesOf(failures, UserCodeRemoteSystemException.class))
.isEqualTo(0L);

pipeline.run();
}

@Test
public void givenCallerThrowsGenericWrappingTimeout_thenPreservesExceptionType() {
Result<Response> result =
pipeline
.apply(Create.of(new Request("a")))
.apply(
Call.of(
new CallerThrowsGenericWrappingTimeout(), NON_DETERMINISTIC_RESPONSE_CODER));

PCollection<ApiIOError> failures = result.getFailures();
PAssert.thatSingleton(countStackTracesOf(failures, UserCodeTimeoutException.class))
.isEqualTo(1L);
PAssert.thatSingleton(countStackTracesOf(failures, UserCodeExecutionException.class))
.isEqualTo(0L);

pipeline.run();
}

@Test
public void givenCallerThrowsGenericWrappingQuota_thenPreservesExceptionType() {
Result<Response> result =
pipeline
.apply(Create.of(new Request("a")))
.apply(
Call.of(new CallerThrowsGenericWrappingQuota(), NON_DETERMINISTIC_RESPONSE_CODER));

PCollection<ApiIOError> failures = result.getFailures();
PAssert.thatSingleton(countStackTracesOf(failures, UserCodeQuotaException.class)).isEqualTo(1L);
PAssert.thatSingleton(countStackTracesOf(failures, UserCodeExecutionException.class))
.isEqualTo(0L);

pipeline.run();
}

@Test
public void givenCallerThrowsGenericWrappingRemoteSystem_thenPreservesExceptionType() {
Result<Response> result =
pipeline
.apply(Create.of(new Request("a")))
.apply(
Call.of(
new CallerThrowsGenericWrappingRemoteSystem(),
NON_DETERMINISTIC_RESPONSE_CODER));

PCollection<ApiIOError> failures = result.getFailures();
PAssert.thatSingleton(countStackTracesOf(failures, UserCodeRemoteSystemException.class))
.isEqualTo(1L);
PAssert.thatSingleton(countStackTracesOf(failures, UserCodeExecutionException.class))
.isEqualTo(0L);

pipeline.run();
}

@Test
public void
givenCallerThrowsUncheckedExecutionExceptionWrappingTimeout_thenPreservesExceptionType() {
Result<Response> result =
pipeline
.apply(Create.of(new Request("a")))
.apply(
Call.of(
new CallerThrowsUncheckedExecutionExceptionWrappingTimeout(),
NON_DETERMINISTIC_RESPONSE_CODER));

PCollection<ApiIOError> failures = result.getFailures();
PAssert.thatSingleton(countStackTracesOf(failures, UserCodeTimeoutException.class))
.isEqualTo(1L);
PAssert.thatSingleton(countStackTracesOf(failures, UserCodeExecutionException.class))
.isEqualTo(0L);

pipeline.run();
}

@Test
public void
givenCallerThrowsUncheckedExecutionExceptionWrappingRemoteSystem_thenPreservesExceptionType() {
Result<Response> result =
pipeline
.apply(Create.of(new Request("a")))
.apply(
Call.of(
new CallerThrowsUncheckedExecutionExceptionWrappingRemoteSystem(),
NON_DETERMINISTIC_RESPONSE_CODER));

PCollection<ApiIOError> failures = result.getFailures();
PAssert.thatSingleton(countStackTracesOf(failures, UserCodeRemoteSystemException.class))
.isEqualTo(1L);
PAssert.thatSingleton(countStackTracesOf(failures, UserCodeExecutionException.class))
.isEqualTo(0L);

pipeline.run();
}

@Test
public void givenCallerThrowsTripleNestedTimeout_thenPreservesExceptionType() {
Result<Response> result =
pipeline
.apply(Create.of(new Request("a")))
.apply(
Call.of(new CallerThrowsTripleNestedTimeout(), NON_DETERMINISTIC_RESPONSE_CODER));

PCollection<ApiIOError> failures = result.getFailures();
PAssert.thatSingleton(countStackTracesOf(failures, UserCodeTimeoutException.class))
.isEqualTo(1L);
PAssert.thatSingleton(countStackTracesOf(failures, UserCodeExecutionException.class))
.isEqualTo(0L);

pipeline.run();
}

@Test
public void givenSetupThrowsUserCodeExecutionException_throwsError() {
pipeline
Expand Down Expand Up @@ -376,6 +551,25 @@ public Response call(Request request) throws UserCodeExecutionException {
}
}

private static class CallerThrowsRuntimeException implements Caller<Request, Response> {

@Override
public Response call(Request request) {
throw new RuntimeException("unexpected error");
}
}

private static class CallerThrowsCircularCause implements Caller<Request, Response> {

@Override
public Response call(Request request) {
Exception a = new Exception("a");
Exception b = new Exception("b", a);
a.initCause(b); // a -> b -> a (circular reference)
throw new RuntimeException("boom", a);
}
}

private static class CallerThrowsTimeout implements Caller<Request, Response> {

@Override
Expand All @@ -384,6 +578,74 @@ public Response call(Request request) throws UserCodeExecutionException {
}
}

private static class CallerThrowsRemoteSystemException implements Caller<Request, Response> {

@Override
public Response call(Request request) throws UserCodeExecutionException {
throw new UserCodeRemoteSystemException("");
}
}

private static class CallerThrowsNestedExecutionException implements Caller<Request, Response> {

@Override
public Response call(Request request) throws UserCodeExecutionException {
throw new UncheckedExecutionException(new UserCodeExecutionException("nested"));
}
}

private static class CallerThrowsGenericWrappingTimeout implements Caller<Request, Response> {

@Override
public Response call(Request request) throws UserCodeExecutionException {
throw new UserCodeExecutionException("generic", new UserCodeTimeoutException("timeout"));
}
}

private static class CallerThrowsGenericWrappingQuota implements Caller<Request, Response> {

@Override
public Response call(Request request) throws UserCodeExecutionException {
throw new UserCodeExecutionException("generic", new UserCodeQuotaException("quota"));
}
}

private static class CallerThrowsGenericWrappingRemoteSystem
implements Caller<Request, Response> {

@Override
public Response call(Request request) throws UserCodeExecutionException {
throw new UserCodeExecutionException("generic", new UserCodeRemoteSystemException("remote"));
}
}

private static class CallerThrowsUncheckedExecutionExceptionWrappingTimeout
implements Caller<Request, Response> {

@Override
public Response call(Request request) throws UserCodeExecutionException {
throw new UncheckedExecutionException(new UserCodeTimeoutException("timeout"));
}
}

private static class CallerThrowsUncheckedExecutionExceptionWrappingRemoteSystem
implements Caller<Request, Response> {

@Override
public Response call(Request request) throws UserCodeExecutionException {
throw new UncheckedExecutionException(new UserCodeRemoteSystemException("remote"));
}
}

private static class CallerThrowsTripleNestedTimeout implements Caller<Request, Response> {

@Override
public Response call(Request request) throws UserCodeExecutionException {
throw new UncheckedExecutionException(
new RuntimeException(new UserCodeTimeoutException("deep timeout")));
}
}

private static class CallerInvokesQuotaException implements Caller<Request, Response> {

@Override
Expand Down
Loading