JNSimba opened a new pull request, #63427:
URL: https://github.com/apache/doris/pull/63427

   ### What problem does this PR solve?
   
   Issue Number: N/A
   
   Related PR: N/A
   
   Problem Summary:
   
   Streaming insert job (CDC source / JdbcSourceOffsetProvider) can become 
permanently stuck in PAUSED when a BE-side commit arrives after FE-side task 
timeout. Symptoms observed in production:
   
   - Job status PAUSED with empty ErrorMsg / JobRuntimeMsg.
   - Latest task PENDING and never scheduled (scheduler logs "do not need to 
schedule invalid task ... job status: PAUSED").
   - Previous task status SUCCESS but its ErrorMsg = "task failed cause 
timeout".
   - `auto resume` never recovers the job; only manual `RESUME JOB` works.
   
   **Root cause**
   
   1. FE `processTimeoutTasks` detects task timeout and calls 
`runningMultiTask.onFail("task failed cause timeout")`. 
`AbstractStreamingTask.onFail` sets task status to FAILED.
   2. `StreamingInsertJob.onStreamTaskFail` sets `failureReason` and calls 
`updateJobStatus(PAUSED)`, which in turn invokes `clearRunningStreamTask` → 
`task.cancel(true)`.
   3. `AbstractStreamingTask.cancel()` short-circuits on terminal status: it 
returns immediately when status is already FAILED/SUCCESS/CANCELED, so 
`isCanceled` is never flipped to `true`.
   4. A late BE callback arrives at `StreamingInsertJob.commitOffset`. The 
current `runningStreamTask != null && instanceof StreamingMultiTblTask + taskId 
match` checks all pass, and downstream defenses in 
`successCallback`/`beforeCommitted` also gate on `getIsCanceled().get()`, which 
is still `false`. `successCallback` therefore overrides task status back to 
SUCCESS, calls `onStreamTaskSuccess` → `resetFailureInfo(null)`, clearing 
`failureReason`.
   5. `StreamingJobSchedulerTask.autoResumeHandler` returns early whenever 
`failureReason == null`, so the PAUSED job is never resumed.
   
   The bug is essentially: `cancel()` is supposed to be the single source of 
truth that says "this task instance is dead, do not accept further callbacks", 
but its terminal short-circuit prevents the signal from being broadcast through 
`isCanceled`, leaving every other defense in the streaming task path silently 
bypassed.
   
   **Fix**
   
   - `AbstractStreamingTask.cancel()`: always flip `isCanceled` on entry, even 
when the task is already in a terminal state. This restores the contract that 
10+ existing `getIsCanceled().get()` checks across the streaming task path rely 
on (e.g. `successCallback`, `beforeCommitted`, internal abort points in 
`StreamingInsertTask` / `StreamingMultiTblTask`).
   - `StreamingInsertJob.commitOffset()`: add an `isCanceled` guard right after 
the `instanceof StreamingMultiTblTask` check so the late callback is dropped 
(logged at INFO) before any side effects (`updateNoTxnJobStatisticAndOffset`, 
`onTaskCommitted`, `persistOffsetProviderIfNeed`) run.
   
   ### Release note
   
   Fix streaming insert job stuck in PAUSED when a late BE commit callback 
arrives after FE-side task timeout.
   
   ### Check List (For Author)
   
   - Test
       - [x] Unit Test
       - [ ] Regression test
       - [ ] Manual test (add detailed scripts or steps below)
       - [ ] No need to test or manual test. Explain why:
           - [ ] This is a refactor/code format and no logic has been changed.
           - [ ] Previous test can cover this change.
           - [ ] No code files have been changed.
           - [ ] Other reason
   
   New unit tests in `StreamingInsertJobLateCallbackTest`:
   - `cancel()` flips `isCanceled` on a terminal-state task (FAILED / SUCCESS) 
without overriding the existing status.
   - `cancel()` transitions a RUNNING task to CANCELED correctly.
   - `cancel()` is idempotent — the second invocation early-returns and leaves 
task state untouched.
   - `commitOffset()` silently skips when the running task is already canceled 
(status preserved, no successCallback side effects).
   
   Regression coverage relies on existing CDC pause/resume suites under 
`regression-test/suites/job_p0/streaming_job/cdc/` to guard the normal happy 
path. The exact "BE late callback after FE timeout" timing cannot be reliably 
reproduced in the existing non-`nonConcurrent` CDC tests without adding debug 
points to `commitOffset`.
   
   - Behavior changed:
       - [x] Yes. A late BE `commitOffset` arriving after a streaming task has 
been canceled is now dropped (logged at INFO) instead of being allowed to 
mutate task / job state and clear `failureReason`. Side effect: on 
non-unique-key target tables, auto-resume may now produce a small number of 
duplicate rows from re-running the same input range, in exchange for the job no 
longer being permanently stuck in PAUSED.
   
   - Does this need documentation?
       - [x] No.


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: [email protected]

For queries about this service, please contact Infrastructure at:
[email protected]


---------------------------------------------------------------------
To unsubscribe, e-mail: [email protected]
For additional commands, e-mail: [email protected]

Reply via email to