diff --git a/gobblin-temporal/src/main/java/org/apache/gobblin/temporal/joblauncher/GobblinTemporalJobLauncher.java b/gobblin-temporal/src/main/java/org/apache/gobblin/temporal/joblauncher/GobblinTemporalJobLauncher.java index 6a07d9def29..863fd058cdc 100644 --- a/gobblin-temporal/src/main/java/org/apache/gobblin/temporal/joblauncher/GobblinTemporalJobLauncher.java +++ b/gobblin-temporal/src/main/java/org/apache/gobblin/temporal/joblauncher/GobblinTemporalJobLauncher.java @@ -137,16 +137,23 @@ protected void executeCancellation() { .build(); DescribeWorkflowExecutionResponse response = workflowServiceStubs.blockingStub().describeWorkflowExecution(request); - // Check if the workflow is not finished - WorkflowExecutionStatus status = response.getWorkflowExecutionInfo().getStatus(); - if (status != WorkflowExecutionStatus.WORKFLOW_EXECUTION_STATUS_COMPLETED && - status != WorkflowExecutionStatus.WORKFLOW_EXECUTION_STATUS_FAILED && - status != WorkflowExecutionStatus.WORKFLOW_EXECUTION_STATUS_CANCELED && - status != WorkflowExecutionStatus.WORKFLOW_EXECUTION_STATUS_TERMINATED) { + try { + // Check if the workflow is not finished + WorkflowExecutionStatus status = response.getWorkflowExecutionInfo().getStatus(); + if (status != WorkflowExecutionStatus.WORKFLOW_EXECUTION_STATUS_COMPLETED && + status != WorkflowExecutionStatus.WORKFLOW_EXECUTION_STATUS_FAILED && + status != WorkflowExecutionStatus.WORKFLOW_EXECUTION_STATUS_CANCELED && + status != WorkflowExecutionStatus.WORKFLOW_EXECUTION_STATUS_TERMINATED) { + workflowStub.cancel(); + log.info("Temporal workflow {} cancelled successfully", this.workflowId); + } else { + log.info("Workflow {} is already finished with status {}", this.workflowId, status); + } + } + catch (Exception e) { + log.warn("Exception occurred while getting status of the workflow " + this.workflowId + + ". We would still attempt the cancellation", e); workflowStub.cancel(); - log.info("Temporal workflow {} cancelled successfully", this.workflowId); - } else { - log.info("Workflow {} is already finished with status {}", this.workflowId, status); } } catch (Exception e) { diff --git a/gobblin-temporal/src/test/java/org/apache/gobblin/temporal/joblauncher/GobblinTemporalJobLauncherTest.java b/gobblin-temporal/src/test/java/org/apache/gobblin/temporal/joblauncher/GobblinTemporalJobLauncherTest.java index af356104818..85bd5a2bb03 100644 --- a/gobblin-temporal/src/test/java/org/apache/gobblin/temporal/joblauncher/GobblinTemporalJobLauncherTest.java +++ b/gobblin-temporal/src/test/java/org/apache/gobblin/temporal/joblauncher/GobblinTemporalJobLauncherTest.java @@ -167,4 +167,16 @@ public void testCancelWorkflowIfRunning() throws Exception { verify(mockStub, times(1)).cancel(); } + + @Test + public void testCancelWorkflowFetchStatusThrowsException() throws Exception { + // Mock the get workflow status to throw an exception + Mockito.doThrow(new RuntimeException("Some exception occurred")).when(mockExecutionInfo).getStatus(); + + jobLauncher.submitJob(null); + + jobLauncher.executeCancellation(); + + verify(mockStub, times(1)).cancel(); + } } \ No newline at end of file