Skip to content

Commit

Permalink
Add error code for Presto-on-Spark
Browse files Browse the repository at this point in the history
  • Loading branch information
wenleix committed Aug 19, 2020
1 parent ad1ab8b commit c02e90e
Show file tree
Hide file tree
Showing 3 changed files with 69 additions and 6 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -126,6 +126,8 @@
import static com.facebook.presto.execution.StageInfo.getAllStages;
import static com.facebook.presto.execution.scheduler.StreamingPlanSection.extractStreamingSections;
import static com.facebook.presto.execution.scheduler.TableWriteInfo.createTableWriteInfo;
import static com.facebook.presto.spark.SparkErrorCode.GENERIC_SPARK_ERROR;
import static com.facebook.presto.spark.SparkErrorCode.SPARK_EXECUTOR_OOM;
import static com.facebook.presto.spark.classloader_interface.ScalaUtils.collectScalaIterator;
import static com.facebook.presto.spark.classloader_interface.ScalaUtils.emptyScalaIterator;
import static com.facebook.presto.spark.util.PrestoSparkUtils.toSerializedPage;
Expand Down Expand Up @@ -625,7 +627,7 @@ public List<List<Object>> execute()
commit(session, transactionManager);
queryStateTimer.endQuery();
}
catch (Exception executionFailure) {
catch (Exception executionException) {
queryStateTimer.beginFinishing();
try {
rollback(session, transactionManager);
Expand All @@ -635,14 +637,29 @@ public List<List<Object>> execute()
}

Optional<ExecutionFailureInfo> failureInfo = Optional.empty();
if (executionFailure instanceof SparkException) {
failureInfo = executionExceptionFactory.extractExecutionFailureInfo((SparkException) executionFailure);
if (executionException instanceof SparkException) {
SparkException sparkException = (SparkException) executionException;
failureInfo = executionExceptionFactory.extractExecutionFailureInfo(sparkException);

if (!failureInfo.isPresent()) {
// not a SparkException with Presto failure info encoded
PrestoException wrappedPrestoException;
if (sparkException.getMessage().contains("most recent failure: JVM_OOM")) {
wrappedPrestoException = new PrestoException(SPARK_EXECUTOR_OOM, executionException);
}
else {
wrappedPrestoException = new PrestoException(GENERIC_SPARK_ERROR, executionException);
}

failureInfo = Optional.of(toFailure(wrappedPrestoException));
}
}
if (!failureInfo.isPresent() && executionFailure instanceof PrestoSparkExecutionException) {
failureInfo = executionExceptionFactory.extractExecutionFailureInfo((PrestoSparkExecutionException) executionFailure);
else if (executionException instanceof PrestoSparkExecutionException) {
failureInfo = executionExceptionFactory.extractExecutionFailureInfo((PrestoSparkExecutionException) executionException);
}

if (!failureInfo.isPresent()) {
failureInfo = Optional.of(toFailure(executionFailure));
failureInfo = Optional.of(toFailure(executionException));
}

queryStateTimer.endQuery();
Expand Down
Original file line number Diff line number Diff line change
@@ -0,0 +1,43 @@
/*
* Licensed under the Apache License, Version 2.0 (the "License");
* you may not use this file except in compliance with the License.
* You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package com.facebook.presto.spark;

import com.facebook.presto.spi.ErrorCode;
import com.facebook.presto.spi.ErrorCodeSupplier;
import com.facebook.presto.spi.ErrorType;

import static com.facebook.presto.spi.ErrorType.INTERNAL_ERROR;

public enum SparkErrorCode
implements ErrorCodeSupplier
{
GENERIC_SPARK_ERROR(0, INTERNAL_ERROR),
SPARK_EXECUTOR_OOM(1, INTERNAL_ERROR),
/**/;

private final ErrorCode errorCode;

public static final int ERROR_CODE_MASK = 0x0003_0000;

SparkErrorCode(int code, ErrorType type)
{
errorCode = new ErrorCode(code + ERROR_CODE_MASK, name(), type);
}

@Override
public ErrorCode toErrorCode()
{
return errorCode;
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -110,6 +110,9 @@ public enum StandardErrorCode
EXCEEDED_SCAN_RAW_BYTES_READ_LIMIT(0x0002_0009, INSUFFICIENT_RESOURCES),
/**/;

// Error code range 0x0003 is reserved for Presto-on-Spark
// See com.facebook.presto.spark.SparkErrorCode

// Connectors can use error codes starting at the range 0x0100_0000
// See https://github.com/prestodb/presto/wiki/Error-Codes

Expand Down

0 comments on commit c02e90e

Please sign in to comment.