ferenc-csaky commented on code in PR #23218: URL: https://github.com/apache/flink/pull/23218#discussion_r1296023473
########## flink-runtime/src/test/java/org/apache/flink/runtime/state/AsyncSnapshotCallableTest.java: ########## @@ -102,24 +103,20 @@ public void testExceptionRun() throws Exception { } testBlocker.unblockSuccessfully(); - try { - task.get(); - Assert.fail(); - } catch (ExecutionException ee) { - Assert.assertEquals(IOException.class, ee.getCause().getClass()); - } + assertThatThrownBy(task::get) + .isInstanceOf(ExecutionException.class) + .hasCauseInstanceOf(IOException.class); runner.join(); - Assert.assertEquals( - Arrays.asList(METHOD_CALL, METHOD_CLEANUP), - testAsyncSnapshotCallable.getInvocationOrder()); + assertThat(testAsyncSnapshotCallable.getInvocationOrder()) + .isEqualTo(Arrays.asList(METHOD_CALL, METHOD_CLEANUP)); Review Comment: Instead of `isEqualTo` I'd use `containsExactly(METHOD_CALL, METHOD_CLEANUP)` ########## flink-runtime/src/test/java/org/apache/flink/runtime/state/AsyncSnapshotCallableTest.java: ########## @@ -156,40 +148,29 @@ public void testCloseRun() throws Exception { ownerRegistry.close(); - try { - task.get(); - Assert.fail(); - } catch (CancellationException ignored) { - } + assertThatThrownBy(task::get).isInstanceOf(CancellationException.class); runner.join(); - Assert.assertEquals( - Arrays.asList(METHOD_CALL, METHOD_CANCEL, METHOD_CLEANUP), - testAsyncSnapshotCallable.getInvocationOrder()); - Assert.assertTrue(testBlocker.isClosed()); + assertThat(testAsyncSnapshotCallable.getInvocationOrder()) + .isEqualTo(Arrays.asList(METHOD_CALL, METHOD_CANCEL, METHOD_CLEANUP)); + assertThat(testBlocker.isClosed()).isTrue(); } @Test - public void testCancelBeforeRun() throws Exception { + void testCancelBeforeRun() throws Exception { task.cancel(true); Thread runner = startTask(task); - try { - task.get(); - Assert.fail(); - } catch (CancellationException ignored) { - } + assertThatThrownBy(task::get).isInstanceOf(CancellationException.class); runner.join(); - Assert.assertEquals( - Arrays.asList(METHOD_CANCEL, METHOD_CLEANUP), - testAsyncSnapshotCallable.getInvocationOrder()); - - Assert.assertTrue(testProvidedResource.isClosed()); + assertThat(testAsyncSnapshotCallable.getInvocationOrder()) + .isEqualTo(Arrays.asList(METHOD_CANCEL, METHOD_CLEANUP)); Review Comment: Same `isEqualTo` -> `containsExactly`. ########## flink-runtime/src/test/java/org/apache/flink/runtime/state/AsyncSnapshotCallableTest.java: ########## @@ -156,40 +148,29 @@ public void testCloseRun() throws Exception { ownerRegistry.close(); - try { - task.get(); - Assert.fail(); - } catch (CancellationException ignored) { - } + assertThatThrownBy(task::get).isInstanceOf(CancellationException.class); runner.join(); - Assert.assertEquals( - Arrays.asList(METHOD_CALL, METHOD_CANCEL, METHOD_CLEANUP), - testAsyncSnapshotCallable.getInvocationOrder()); - Assert.assertTrue(testBlocker.isClosed()); + assertThat(testAsyncSnapshotCallable.getInvocationOrder()) + .isEqualTo(Arrays.asList(METHOD_CALL, METHOD_CANCEL, METHOD_CLEANUP)); Review Comment: Same `isEqualTo` -> `containsExactly`. ########## flink-runtime/src/test/java/org/apache/flink/runtime/state/StateBackendTestBase.java: ########## @@ -355,44 +336,42 @@ public void testKeyGroupedInternalPriorityQueue(boolean addAll) throws Exception if (addAll) { priorityQueue.addAll(asList(elements)); } else { - assertTrue(priorityQueue.add(elements[0])); - assertTrue(priorityQueue.add(elements[1])); - assertFalse(priorityQueue.add(elements[2])); - assertFalse(priorityQueue.add(elements[3])); - assertFalse(priorityQueue.add(elements[4])); + assertThat(priorityQueue.add(elements[0])).isTrue(); + assertThat(priorityQueue.add(elements[1])).isTrue(); + assertThat(priorityQueue.add(elements[2])).isFalse(); + assertThat(priorityQueue.add(elements[3])).isFalse(); + assertThat(priorityQueue.add(elements[4])).isFalse(); } - assertFalse(priorityQueue.isEmpty()); - assertThat( - priorityQueue.getSubsetForKeyGroup(1), - containsInAnyOrder(elementA42, elementA44)); - assertThat( - priorityQueue.getSubsetForKeyGroup(8), - containsInAnyOrder(elementB1, elementB3)); + assertThat(priorityQueue.isEmpty()).isFalse(); Review Comment: i'd use `assertThat(priorityQueue).isNotEmpty()`. ########## flink-runtime/src/test/java/org/apache/flink/runtime/state/filesystem/CheckpointStateOutputStreamTest.java: ########## @@ -45,40 +46,36 @@ import java.util.Random; import java.util.UUID; -import static org.junit.Assert.assertArrayEquals; -import static org.junit.Assert.assertEquals; -import static org.junit.Assert.assertFalse; -import static org.junit.Assert.assertNotNull; -import static org.junit.Assert.assertTrue; -import static org.junit.Assert.fail; +import static org.assertj.core.api.Assertions.assertThat; +import static org.assertj.core.api.Assertions.assertThatThrownBy; import static org.mockito.Mockito.spy; import static org.mockito.Mockito.verify; /** Abstract base class for tests against checkpointing streams. */ -@RunWith(Parameterized.class) -public class CheckpointStateOutputStreamTest extends TestLogger { +@ExtendWith(ParameterizedTestExtension.class) +public class CheckpointStateOutputStreamTest { Review Comment: The inner `enum` can be `private`. ########## flink-runtime/src/test/java/org/apache/flink/runtime/state/heap/HeapStateBackendTestBase.java: ########## Review Comment: This class is completely unused. Any reason to keep it? ########## flink-state-backends/flink-statebackend-rocksdb/src/test/java/org/apache/flink/contrib/streaming/state/ttl/RocksDBTtlStateTestBase.java: ########## Review Comment: Class still uses `Junit4` asserts instead of `AssertJ`. ########## flink-runtime/src/test/java/org/apache/flink/runtime/state/AsyncSnapshotCallableTest.java: ########## @@ -130,23 +127,18 @@ public void testCancelRun() throws Exception { task.cancel(true); testBlocker.unblockExceptionally(); - try { - task.get(); - Assert.fail(); - } catch (CancellationException ignored) { - } + assertThatThrownBy(task::get).isInstanceOf(CancellationException.class); runner.join(); - Assert.assertEquals( - Arrays.asList(METHOD_CALL, METHOD_CANCEL, METHOD_CLEANUP), - testAsyncSnapshotCallable.getInvocationOrder()); - Assert.assertTrue(testProvidedResource.isClosed()); - Assert.assertTrue(testBlocker.isClosed()); + assertThat(testAsyncSnapshotCallable.getInvocationOrder()) + .isEqualTo(Arrays.asList(METHOD_CALL, METHOD_CANCEL, METHOD_CLEANUP)); Review Comment: Same `isEqualTo` -> `containsExactly`. ########## flink-runtime/src/test/java/org/apache/flink/runtime/state/FileStateBackendTest.java: ########## @@ -65,23 +68,23 @@ protected boolean supportsAsynchronousSnapshots() { // disable these because the verification does not work for this state backend @Override - @Test + @TestTemplate Review Comment: Test methods can be package-private, except the last one which is defined as protected. ########## flink-runtime/src/test/java/org/apache/flink/runtime/state/StateBackendTestBase.java: ########## @@ -957,25 +951,42 @@ public void testKryoRegisteringRestoreResilienceWithDefaultSerializer() throws E .addDefaultKryoSerializer( TestPojo.class, (Class) CustomKryoTestSerializer.class); - // on the second restore, since the custom serializer will be used for - // deserialization, we expect the deliberate failure to be thrown - expectedException.expect( - anyOf( - isA(ExpectedKryoTestException.class), - Matchers.<Throwable>hasProperty( - "cause", isA(ExpectedKryoTestException.class)))); - - // state backends that eagerly deserializes (such as the memory state backend) will fail - // here - backend = restoreKeyedBackend(IntSerializer.INSTANCE, snapshot2, env); - - state = - backend.getPartitionedState( - VoidNamespace.INSTANCE, VoidNamespaceSerializer.INSTANCE, kvId); - - backend.setCurrentKey(1); - // state backends that lazily deserializes (such as RocksDB) will fail here - state.value(); + assertThatThrownBy( + () -> { + // state backends that eagerly deserializes (such as the memory + // state backend) will fail here + CheckpointableKeyedStateBackend<Integer> restoreBackend = null; + try { + restoreBackend = + restoreKeyedBackend( + IntSerializer.INSTANCE, snapshot2, env); + + ValueState<TestPojo> restoreState = + restoreBackend.getPartitionedState( + VoidNamespace.INSTANCE, + VoidNamespaceSerializer.INSTANCE, + new ValueStateDescriptor<>("id", pojoType)); Review Comment: IMO initializing a new `final` variable with `kvId` before the `assertThrownBy` call and using that inside would be better in this case, because if this value changes, now we it has to be updated in 2 places. ########## flink-runtime/src/test/java/org/apache/flink/runtime/state/TaskStateManagerImplTest.java: ########## @@ -158,46 +157,48 @@ public void testStateReportingAndRetrieving() { PrioritizedOperatorSubtaskState prioritized_3 = taskStateManager.prioritizedOperatorState(operatorID_3); - Assert.assertTrue(prioritized_1.isRestored()); - Assert.assertTrue(prioritized_2.isRestored()); - Assert.assertTrue(prioritized_3.isRestored()); - Assert.assertTrue(taskStateManager.prioritizedOperatorState(new OperatorID()).isRestored()); + assertThat(prioritized_1.isRestored()).isTrue(); + assertThat(prioritized_2.isRestored()).isTrue(); + assertThat(prioritized_3.isRestored()).isTrue(); + assertThat(taskStateManager.prioritizedOperatorState(new OperatorID()).isRestored()) + .isTrue(); // checks for operator 1. Iterator<StateObjectCollection<KeyedStateHandle>> prioritizedManagedKeyedState_1 = prioritized_1.getPrioritizedManagedKeyedState().iterator(); - Assert.assertTrue(prioritizedManagedKeyedState_1.hasNext()); + assertThat(prioritizedManagedKeyedState_1.hasNext()).isTrue(); Review Comment: For iterators, it is possible to `assertThat(prioritizedManagedKeyedState_1).hasNext()`, I suggest to use that. Same for every other `hasNext` assert in file. There is no negate function for the shorthand though, so if the iterator should not have a next elem, the `assertThat(...).isFalse()` format is the only option, like @ line 181. ########## flink-runtime/src/test/java/org/apache/flink/runtime/state/StateBackendTestBase.java: ########## @@ -355,44 +336,42 @@ public void testKeyGroupedInternalPriorityQueue(boolean addAll) throws Exception if (addAll) { priorityQueue.addAll(asList(elements)); } else { - assertTrue(priorityQueue.add(elements[0])); - assertTrue(priorityQueue.add(elements[1])); - assertFalse(priorityQueue.add(elements[2])); - assertFalse(priorityQueue.add(elements[3])); - assertFalse(priorityQueue.add(elements[4])); + assertThat(priorityQueue.add(elements[0])).isTrue(); + assertThat(priorityQueue.add(elements[1])).isTrue(); + assertThat(priorityQueue.add(elements[2])).isFalse(); + assertThat(priorityQueue.add(elements[3])).isFalse(); + assertThat(priorityQueue.add(elements[4])).isFalse(); } - assertFalse(priorityQueue.isEmpty()); - assertThat( - priorityQueue.getSubsetForKeyGroup(1), - containsInAnyOrder(elementA42, elementA44)); - assertThat( - priorityQueue.getSubsetForKeyGroup(8), - containsInAnyOrder(elementB1, elementB3)); + assertThat(priorityQueue.isEmpty()).isFalse(); + assertThat(priorityQueue.getSubsetForKeyGroup(1)) + .containsExactlyInAnyOrder(elementA42, elementA44); + assertThat(priorityQueue.getSubsetForKeyGroup(8)) + .containsExactlyInAnyOrder(elementB1, elementB3); - assertThat(priorityQueue.peek(), equalTo(elementB1)); - assertThat(priorityQueue.poll(), equalTo(elementB1)); - assertThat(priorityQueue.peek(), equalTo(elementB3)); + assertThat(priorityQueue.peek()).isEqualTo(elementB1); + assertThat(priorityQueue.poll()).isEqualTo(elementB1); + assertThat(priorityQueue.peek()).isEqualTo(elementB3); List<TestType> actualList = new ArrayList<>(); try (CloseableIterator<TestType> iterator = priorityQueue.iterator()) { iterator.forEachRemaining(actualList::add); } - assertThat(actualList, containsInAnyOrder(elementB3, elementA42, elementA44)); + assertThat(actualList).containsExactlyInAnyOrder(elementB3, elementA42, elementA44); - assertEquals(3, priorityQueue.size()); + assertThat(priorityQueue.size()).isEqualTo(3); Review Comment: Shorthand for size: `assertThat(priorityQueue).hasSize(3)`. ########## flink-runtime/src/test/java/org/apache/flink/runtime/state/StateBackendTestBase.java: ########## @@ -1078,25 +1089,41 @@ public void testKryoRegisteringRestoreResilienceWithRegisteredSerializer() throw env.getExecutionConfig() .registerTypeWithKryoSerializer(TestPojo.class, CustomKryoTestSerializer.class); - // on the second restore, since the custom serializer will be used for - // deserialization, we expect the deliberate failure to be thrown - expectedException.expect( - anyOf( - isA(ExpectedKryoTestException.class), - Matchers.<Throwable>hasProperty( - "cause", isA(ExpectedKryoTestException.class)))); - - // state backends that eagerly deserializes (such as the memory state backend) will fail - // here - backend = restoreKeyedBackend(IntSerializer.INSTANCE, snapshot2, env); - - state = - backend.getPartitionedState( - VoidNamespace.INSTANCE, VoidNamespaceSerializer.INSTANCE, kvId); + assertThatThrownBy( Review Comment: Logic in the `assertThatThrownBy` seems to be the same as we have @ line 954-989. I think this could be extracted into a separate assert method without adding complexity, so we do not duplicate this code. ########## flink-runtime/src/test/java/org/apache/flink/runtime/state/StateBackendTestBase.java: ########## @@ -431,11 +410,11 @@ public void testGetKeys() throws Exception { keysStream.mapToInt(value -> value.intValue()).iterator(); for (int expectedKey = 0; expectedKey < namespace1ElementsNum; expectedKey++) { - assertTrue(actualIterator.hasNext()); - assertEquals(expectedKey, actualIterator.nextInt()); + assertThat(actualIterator.hasNext()).isTrue(); Review Comment: Shorthand for iter: `assertThat(actualIterator).hasNext()` ########## flink-runtime/src/test/java/org/apache/flink/runtime/state/HashMapStateBackendTest.java: ########## @@ -89,23 +90,23 @@ protected boolean isSerializerPresenceRequiredOnRestore() { // disable these because the verification does not work for this state backend @Override - @Test + @TestTemplate Review Comment: Public visibility can be lovered for test methods. ########## flink-runtime/src/test/java/org/apache/flink/runtime/state/StateBackendTestBase.java: ########## @@ -5047,11 +5078,11 @@ public void testMapStateGetKeys() throws Exception { keysStream.mapToInt(value -> value.intValue()).iterator(); for (int expectedKey = 0; expectedKey < namespace1ElementsNum; expectedKey++) { - assertTrue(actualIterator.hasNext()); - assertEquals(expectedKey, actualIterator.nextInt()); + assertThat(actualIterator.hasNext()).isTrue(); Review Comment: `assertThat(actualIterator).hasNext()` shorthand. Same @ L5096 ########## flink-runtime/src/test/java/org/apache/flink/runtime/state/TaskStateManagerImplTest.java: ########## @@ -35,25 +35,25 @@ import org.apache.flink.runtime.state.changelog.inmemory.InMemoryStateChangelogStorage; import org.apache.flink.runtime.taskmanager.CheckpointResponder; import org.apache.flink.runtime.taskmanager.TestCheckpointResponder; -import org.apache.flink.util.TestLogger; +import org.apache.flink.testutils.junit.utils.TempDirUtils; import org.apache.flink.util.concurrent.Executors; -import org.junit.Assert; -import org.junit.Test; -import org.junit.rules.TemporaryFolder; +import org.junit.jupiter.api.Test; +import org.junit.jupiter.api.io.TempDir; import java.io.File; -import java.io.IOException; +import java.nio.file.Path; import java.util.Iterator; import java.util.concurrent.Executor; import static org.apache.flink.runtime.executiongraph.ExecutionGraphTestUtils.createExecutionAttemptId; +import static org.assertj.core.api.Assertions.assertThat; -public class TaskStateManagerImplTest extends TestLogger { +public class TaskStateManagerImplTest { Review Comment: IMO it would worth to remove the `static` helper function from this class to a `public` test util class, so this one can be package-private. ########## flink-runtime/src/test/java/org/apache/flink/runtime/state/KeyGroupPartitionerTestBase.java: ########## @@ -68,7 +64,7 @@ protected KeyGroupPartitionerTestBase( } @Test - public void testPartitionByKeyGroup() throws IOException { + void testPartitionByKeyGroup() throws IOException { Review Comment: The other test method can be package-private too. ########## flink-runtime/src/test/java/org/apache/flink/runtime/state/MemoryStateBackendTest.java: ########## @@ -56,23 +58,23 @@ protected boolean supportsAsynchronousSnapshots() { // disable these because the verification does not work for this state backend @Override - @Test + @TestTemplate Review Comment: Public visibility can be lovered for test methods. ########## flink-runtime/src/test/java/org/apache/flink/runtime/state/filesystem/FsCheckpointStorageAccessTest.java: ########## @@ -233,11 +229,11 @@ public void testStorageLocationMkdirs() throws Exception { WRITE_BUFFER_SIZE); File baseDir = new File(storage.getCheckpointsDirectory().getPath()); - assertFalse(baseDir.exists()); + assertThat(baseDir.exists()).isFalse(); Review Comment: `File` assert shorthands can be used. Same for other occurences in current file. ########## flink-runtime/src/test/java/org/apache/flink/runtime/state/OperatorStateBackendTest.java: ########## @@ -640,47 +627,48 @@ public void testSnapshotRestoreSync() throws Exception { broadcastState2 = operatorStateBackend.getBroadcastState(broadcastStateDescriptor2); broadcastState3 = operatorStateBackend.getBroadcastState(broadcastStateDescriptor3); - assertEquals(3, operatorStateBackend.getRegisteredStateNames().size()); - assertEquals(3, operatorStateBackend.getRegisteredBroadcastStateNames().size()); + assertThat(operatorStateBackend.getRegisteredStateNames()).hasSize(3); + assertThat(operatorStateBackend.getRegisteredBroadcastStateNames()).hasSize(3); Iterator<Serializable> it = listState1.get().iterator(); - assertEquals(42, it.next()); - assertEquals(4711, it.next()); - assertFalse(it.hasNext()); + assertThat(it.next()).isEqualTo(42); + assertThat(it.next()).isEqualTo(4711); + assertThat(it.hasNext()).isFalse(); it = listState2.get().iterator(); - assertEquals(7, it.next()); - assertEquals(13, it.next()); - assertEquals(23, it.next()); - assertFalse(it.hasNext()); + assertThat(it.next()).isEqualTo(7); + assertThat(it.next()).isEqualTo(13); + assertThat(it.next()).isEqualTo(23); + assertThat(it.hasNext()).isFalse(); it = listState3.get().iterator(); - assertEquals(17, it.next()); - assertEquals(18, it.next()); - assertEquals(19, it.next()); - assertEquals(20, it.next()); - assertFalse(it.hasNext()); + assertThat(it.next()).isEqualTo(17); + assertThat(it.next()).isEqualTo(18); + assertThat(it.next()).isEqualTo(19); + assertThat(it.next()).isEqualTo(20); + assertThat(it.hasNext()).isFalse(); Iterator<Map.Entry<Serializable, Serializable>> bIt = broadcastState1.iterator(); - assertTrue(bIt.hasNext()); + assertThat(bIt.hasNext()).isTrue(); Review Comment: `assertThat(bIt).hasNext()` shorthand can be used. Same for other occurences in file. ########## flink-runtime/src/test/java/org/apache/flink/runtime/state/filesystem/FsCheckpointStorageAccessTest.java: ########## Review Comment: The class itself can be package-private. ########## flink-runtime/src/test/java/org/apache/flink/runtime/state/memory/MemoryCheckpointStorageAccessTest.java: ########## @@ -200,16 +194,16 @@ public void testTaskOwnedStateStream() throws Exception { * initializeLocationForCheckpoint. */ @Test - public void testStorageLocationMkdirs() throws Exception { + void testStorageLocationMkdirs() throws Exception { MemoryBackendCheckpointStorageAccess storage = new MemoryBackendCheckpointStorageAccess( new JobID(), randomTempPath(), null, DEFAULT_MAX_STATE_SIZE); File baseDir = new File(storage.getCheckpointsDirectory().getPath()); - assertFalse(baseDir.exists()); + assertThat(baseDir.exists()).isFalse(); Review Comment: File assert shorthands can be used. Same for L207. ########## flink-runtime/src/test/java/org/apache/flink/runtime/state/filesystem/FsCheckpointStateOutputStreamTest.java: ########## @@ -399,20 +388,16 @@ public void testStreamDoesNotTryToCleanUpParentOnError() throws Exception { stream1.write(new byte[61]); stream2.write(new byte[61]); - try { - stream1.closeAndGetHandle(); - fail("this should fail with an exception"); - } catch (IOException ignored) { - } + assertThatThrownBy(stream1::closeAndGetHandle).isInstanceOf(IOException.class); stream2.close(); // no delete call must have happened verify(fs, times(0)).delete(any(Path.class), anyBoolean()); // the directory must still exist as a proper directory - assertTrue(directory.exists()); - assertTrue(directory.isDirectory()); + assertThat(directory.exists()).isTrue(); + assertThat(directory.isDirectory()).isTrue(); Review Comment: File assert shorthand can be used: ```java assertThat(directory).exists(); assertThat(directory).isDirectory(); ``` -- 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: issues-unsubscr...@flink.apache.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org