mirror of https://github.com/apache/kafka.git
rewrite to test same as previous
This commit is contained in:
parent
9daf9cdb0f
commit
06b65dc5d3
|
@ -2282,31 +2282,41 @@ public class TaskManagerTest {
|
||||||
}
|
}
|
||||||
|
|
||||||
@Test
|
@Test
|
||||||
public void shouldNotCommitCorruptedTasksOnTaskCorruptedException() {
|
public void shouldCommitNonCorruptedTasksOnTaskCorruptedException() {
|
||||||
final StreamTask corruptedTask = statefulTask(taskId00, taskId00ChangelogPartitions)
|
final StreamTask corruptedTask = statefulTask(taskId00, taskId00ChangelogPartitions)
|
||||||
.withInputPartitions(taskId00Partitions)
|
.withInputPartitions(taskId00Partitions)
|
||||||
.inState(State.RUNNING)
|
.inState(State.RUNNING)
|
||||||
.build();
|
.build();
|
||||||
|
|
||||||
final TaskCorruptedException corruptedException = new TaskCorruptedException(singleton(taskId00));
|
final StreamTask nonCorruptedTask = statefulTask(taskId01, taskId01ChangelogPartitions)
|
||||||
final ExceptionAndTask exceptionAndTask = new ExceptionAndTask(corruptedException, corruptedTask);
|
.withInputPartitions(taskId01Partitions)
|
||||||
|
.inState(State.RUNNING)
|
||||||
when(stateUpdater.hasExceptionsAndFailedTasks()).thenReturn(true);
|
.build();
|
||||||
when(stateUpdater.drainExceptionsAndFailedTasks()).thenReturn(singletonList(exceptionAndTask));
|
|
||||||
|
|
||||||
final TasksRegistry tasks = mock(TasksRegistry.class);
|
final TasksRegistry tasks = mock(TasksRegistry.class);
|
||||||
|
when(tasks.task(taskId00)).thenReturn(corruptedTask);
|
||||||
|
when(tasks.allTasksPerId()).thenReturn(mkMap(
|
||||||
|
mkEntry(taskId00, corruptedTask),
|
||||||
|
mkEntry(taskId01, nonCorruptedTask)
|
||||||
|
));
|
||||||
|
when(tasks.activeTaskIds()).thenReturn(Set.of(taskId00, taskId01));
|
||||||
|
|
||||||
|
when(nonCorruptedTask.commitNeeded()).thenReturn(true);
|
||||||
|
when(nonCorruptedTask.prepareCommit(true)).thenReturn(emptyMap());
|
||||||
|
when(corruptedTask.prepareCommit(false)).thenReturn(emptyMap());
|
||||||
|
doNothing().when(corruptedTask).postCommit(anyBoolean());
|
||||||
|
|
||||||
|
when(consumer.assignment()).thenReturn(taskId00Partitions);
|
||||||
|
|
||||||
final TaskManager taskManager = setUpTaskManager(ProcessingMode.AT_LEAST_ONCE, tasks, false);
|
final TaskManager taskManager = setUpTaskManager(ProcessingMode.AT_LEAST_ONCE, tasks, false);
|
||||||
|
|
||||||
final TaskCorruptedException thrown = assertThrows(
|
taskManager.handleCorruption(Set.of(taskId00));
|
||||||
TaskCorruptedException.class,
|
|
||||||
() -> taskManager.checkStateUpdater(time.milliseconds(), noOpResetter)
|
|
||||||
);
|
|
||||||
|
|
||||||
assertEquals(singleton(taskId00), thrown.corruptedTasks());
|
verify(nonCorruptedTask).prepareCommit(true);
|
||||||
assertEquals("Tasks [0_0] are corrupted and hence need to be re-initialized", thrown.getMessage());
|
verify(nonCorruptedTask, never()).addPartitionsForOffsetReset(any());
|
||||||
|
verify(corruptedTask).addPartitionsForOffsetReset(taskId00Partitions);
|
||||||
verify(tasks).addFailedTask(corruptedTask);
|
verify(corruptedTask).changelogPartitions();
|
||||||
|
verify(corruptedTask).postCommit(true);
|
||||||
|
|
||||||
// check that we should not commit empty map either
|
// check that we should not commit empty map either
|
||||||
verify(consumer, never()).commitSync(emptyMap());
|
verify(consumer, never()).commitSync(emptyMap());
|
||||||
|
|
Loading…
Reference in New Issue