13
13
import org .elasticsearch .ResourceNotFoundException ;
14
14
import org .elasticsearch .action .ActionListener ;
15
15
import org .elasticsearch .action .index .IndexRequest ;
16
- import org .elasticsearch .action .search .SearchPhaseExecutionException ;
17
16
import org .elasticsearch .action .search .SearchRequest ;
18
17
import org .elasticsearch .action .search .SearchResponse ;
19
- import org .elasticsearch .action .search .ShardSearchFailure ;
20
18
import org .elasticsearch .action .support .IndicesOptions ;
21
19
import org .elasticsearch .common .Strings ;
22
20
import org .elasticsearch .common .breaker .CircuitBreakingException ;
23
21
import org .elasticsearch .common .xcontent .XContentBuilder ;
24
22
import org .elasticsearch .index .IndexNotFoundException ;
25
23
import org .elasticsearch .index .query .BoolQueryBuilder ;
26
24
import org .elasticsearch .index .query .QueryBuilder ;
25
+ import org .elasticsearch .script .ScriptException ;
27
26
import org .elasticsearch .search .aggregations .Aggregations ;
28
27
import org .elasticsearch .search .aggregations .bucket .composite .CompositeAggregation ;
29
28
import org .elasticsearch .search .aggregations .bucket .composite .CompositeAggregationBuilder ;
45
44
import org .elasticsearch .xpack .transform .persistence .TransformConfigManager ;
46
45
import org .elasticsearch .xpack .transform .transforms .pivot .AggregationResultUtils ;
47
46
import org .elasticsearch .xpack .transform .transforms .pivot .Pivot ;
47
+ import org .elasticsearch .xpack .transform .utils .ExceptionRootCauseFinder ;
48
48
49
49
import java .io .IOException ;
50
50
import java .io .UncheckedIOException ;
@@ -66,7 +66,7 @@ public abstract class TransformIndexer extends AsyncTwoPhaseIndexer<TransformInd
66
66
* which query filters to run and which index requests to send
67
67
*/
68
68
private enum RunState {
69
- // do a complete query/index, this is used for batch data frames and for bootstraping (1st run)
69
+ // do a complete query/index, this is used for batch transforms and for bootstrapping (1st run)
70
70
FULL_RUN ,
71
71
72
72
// Partial run modes in 2 stages:
@@ -422,7 +422,7 @@ protected IterationResult<TransformIndexerPosition> doProcess(SearchResponse sea
422
422
default :
423
423
// Any other state is a bug, should not happen
424
424
logger .warn ("[{}] Encountered unexpected run state [{}]" , getJobId (), runState );
425
- throw new IllegalStateException ("DataFrame indexer job encountered an illegal state [" + runState + "]" );
425
+ throw new IllegalStateException ("Transform indexer job encountered an illegal state [" + runState + "]" );
426
426
}
427
427
}
428
428
@@ -468,25 +468,36 @@ protected void onAbort() {
468
468
469
469
synchronized void handleFailure (Exception e ) {
470
470
logger .warn (new ParameterizedMessage ("[{}] transform encountered an exception: " , getJobId ()), e );
471
- if (handleCircuitBreakingException (e )) {
472
- return ;
473
- }
474
-
475
- if (isIrrecoverableFailure (e ) || context .getAndIncrementFailureCount () > context .getNumFailureRetries ()) {
476
- String failureMessage = isIrrecoverableFailure (e )
477
- ? "task encountered irrecoverable failure: " + e .getMessage ()
478
- : "task encountered more than " + context .getNumFailureRetries () + " failures; latest failure: " + e .getMessage ();
479
- failIndexer (failureMessage );
471
+ Throwable unwrappedException = ExceptionRootCauseFinder .getRootCauseException (e );
472
+
473
+ if (unwrappedException instanceof CircuitBreakingException ) {
474
+ handleCircuitBreakingException ((CircuitBreakingException ) unwrappedException );
475
+ } else if (unwrappedException instanceof ScriptException ) {
476
+ handleScriptException ((ScriptException ) unwrappedException );
477
+ // irrecoverable error without special handling
478
+ } else if (unwrappedException instanceof IndexNotFoundException
479
+ || unwrappedException instanceof AggregationResultUtils .AggregationExtractionException
480
+ || unwrappedException instanceof TransformConfigReloadingException ) {
481
+ failIndexer ("task encountered irrecoverable failure: " + e .getMessage ());
482
+ } else if (context .getAndIncrementFailureCount () > context .getNumFailureRetries ()) {
483
+ failIndexer (
484
+ "task encountered more than "
485
+ + context .getNumFailureRetries ()
486
+ + " failures; latest failure: "
487
+ + ExceptionRootCauseFinder .getDetailedMessage (unwrappedException )
488
+ );
480
489
} else {
481
490
// Since our schedule fires again very quickly after failures it is possible to run into the same failure numerous
482
491
// times in a row, very quickly. We do not want to spam the audit log with repeated failures, so only record the first one
483
492
if (e .getMessage ().equals (lastAuditedExceptionMessage ) == false ) {
493
+ String message = ExceptionRootCauseFinder .getDetailedMessage (unwrappedException );
494
+
484
495
auditor
485
496
.warning (
486
497
getJobId (),
487
- "Transform encountered an exception: " + e . getMessage () + " Will attempt again at next scheduled trigger."
498
+ "Transform encountered an exception: " + message + " Will attempt again at next scheduled trigger."
488
499
);
489
- lastAuditedExceptionMessage = e . getMessage () ;
500
+ lastAuditedExceptionMessage = message ;
490
501
}
491
502
}
492
503
}
@@ -510,12 +521,6 @@ private void sourceHasChanged(ActionListener<Boolean> hasChangedListener) {
510
521
}));
511
522
}
512
523
513
- private boolean isIrrecoverableFailure (Exception e ) {
514
- return e instanceof IndexNotFoundException
515
- || e instanceof AggregationResultUtils .AggregationExtractionException
516
- || e instanceof TransformConfigReloadingException ;
517
- }
518
-
519
524
private IterationResult <TransformIndexerPosition > processBuckets (final CompositeAggregation agg ) {
520
525
// we reached the end
521
526
if (agg .getBuckets ().isEmpty ()) {
@@ -536,7 +541,7 @@ private IterationResult<TransformIndexerPosition> processBuckets(final Composite
536
541
agg .getBuckets ().isEmpty ()
537
542
);
538
543
539
- // NOTE: progress is also mutated in ClientDataFrameIndexer#onFinished
544
+ // NOTE: progress is also mutated in onFinish
540
545
if (progress != null ) {
541
546
progress .incrementDocsProcessed (getStats ().getNumDocuments () - docsBeforeProcess );
542
547
progress .incrementDocsIndexed (result .getToIndex ().size ());
@@ -671,7 +676,7 @@ protected SearchRequest buildSearchRequest() {
671
676
default :
672
677
// Any other state is a bug, should not happen
673
678
logger .warn ("Encountered unexpected run state [" + runState + "]" );
674
- throw new IllegalStateException ("DataFrame indexer job encountered an illegal state [" + runState + "]" );
679
+ throw new IllegalStateException ("Transform indexer job encountered an illegal state [" + runState + "]" );
675
680
}
676
681
677
682
searchRequest .source (sourceBuilder );
@@ -756,16 +761,9 @@ private SearchSourceBuilder buildPartialUpdateQuery(SearchSourceBuilder sourceBu
756
761
* Implementation details: We take the values from the circuit breaker as a hint, but
757
762
* note that it breaks early, that's why we also reduce using
758
763
*
759
- * @param e Exception thrown, only {@link CircuitBreakingException} are handled
760
- * @return true if exception was handled, false if not
764
+ * @param circuitBreakingException CircuitBreakingException thrown
761
765
*/
762
- protected boolean handleCircuitBreakingException (Exception e ) {
763
- CircuitBreakingException circuitBreakingException = getCircuitBreakingException (e );
764
-
765
- if (circuitBreakingException == null ) {
766
- return false ;
767
- }
768
-
766
+ private void handleCircuitBreakingException (CircuitBreakingException circuitBreakingException ) {
769
767
double reducingFactor = Math
770
768
.min (
771
769
(double ) circuitBreakingException .getByteLimit () / circuitBreakingException .getBytesWanted (),
@@ -777,15 +775,29 @@ protected boolean handleCircuitBreakingException(Exception e) {
777
775
if (newPageSize < MINIMUM_PAGE_SIZE ) {
778
776
String message = TransformMessages .getMessage (TransformMessages .LOG_TRANSFORM_PIVOT_LOW_PAGE_SIZE_FAILURE , pageSize );
779
777
failIndexer (message );
780
- return true ;
778
+ return ;
781
779
}
782
780
783
781
String message = TransformMessages .getMessage (TransformMessages .LOG_TRANSFORM_PIVOT_REDUCE_PAGE_SIZE , pageSize , newPageSize );
784
782
auditor .info (getJobId (), message );
785
- logger .info ("Data frame transform [" + getJobId () + "]:" + message );
786
-
783
+ logger .info ("[{}] {}" , getJobId (), message );
787
784
pageSize = newPageSize ;
788
- return true ;
785
+ return ;
786
+ }
787
+
788
+ /**
789
+ * Handle script exception case. This is error is irrecoverable.
790
+ *
791
+ * @param scriptException ScriptException thrown
792
+ */
793
+ private void handleScriptException (ScriptException scriptException ) {
794
+ String message = TransformMessages
795
+ .getMessage (
796
+ TransformMessages .LOG_TRANSFORM_PIVOT_SCRIPT_ERROR ,
797
+ scriptException .getDetailedMessage (),
798
+ scriptException .getScriptStack ()
799
+ );
800
+ failIndexer (message );
789
801
}
790
802
791
803
protected void failIndexer (String failureMessage ) {
@@ -818,7 +830,7 @@ protected boolean shouldAuditOnFinish(long completedCheckpoint) {
818
830
}
819
831
820
832
private RunState determineRunStateAtStart () {
821
- // either 1st run or not a continuous data frame
833
+ // either 1st run or not a continuous transform
822
834
if (nextCheckpoint .getCheckpoint () == 1 || isContinuous () == false ) {
823
835
return RunState .FULL_RUN ;
824
836
}
@@ -832,32 +844,6 @@ private RunState determineRunStateAtStart() {
832
844
return RunState .PARTIAL_RUN_IDENTIFY_CHANGES ;
833
845
}
834
846
835
- /**
836
- * Inspect exception for circuit breaking exception and return the first one it can find.
837
- *
838
- * @param e Exception
839
- * @return CircuitBreakingException instance if found, null otherwise
840
- */
841
- private static CircuitBreakingException getCircuitBreakingException (Exception e ) {
842
- // circuit breaking exceptions are at the bottom
843
- Throwable unwrappedThrowable = org .elasticsearch .ExceptionsHelper .unwrapCause (e );
844
-
845
- if (unwrappedThrowable instanceof CircuitBreakingException ) {
846
- return (CircuitBreakingException ) unwrappedThrowable ;
847
- } else if (unwrappedThrowable instanceof SearchPhaseExecutionException ) {
848
- SearchPhaseExecutionException searchPhaseException = (SearchPhaseExecutionException ) e ;
849
- for (ShardSearchFailure shardFailure : searchPhaseException .shardFailures ()) {
850
- Throwable unwrappedShardFailure = org .elasticsearch .ExceptionsHelper .unwrapCause (shardFailure .getCause ());
851
-
852
- if (unwrappedShardFailure instanceof CircuitBreakingException ) {
853
- return (CircuitBreakingException ) unwrappedShardFailure ;
854
- }
855
- }
856
- }
857
-
858
- return null ;
859
- }
860
-
861
847
static class TransformConfigReloadingException extends ElasticsearchException {
862
848
TransformConfigReloadingException (String msg , Throwable cause , Object ... args ) {
863
849
super (msg , cause , args );
0 commit comments