From 4438179c6b2181fcaec452f16a52a2b9033dc706 Mon Sep 17 00:00:00 2001 From: Yuxin Tan Date: Tue, 14 Mar 2023 10:40:43 +0800 Subject: [PATCH] [FLINK-31418][network][tests] Fix buffer request timeout for sort merge result partition read test --- .../partition/SortMergeResultPartitionReadSchedulerTest.java | 4 ++-- 1 file changed, 2 insertions(+), 2 deletions(-) diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/io/network/partition/SortMergeResultPartitionReadSchedulerTest.java b/flink-runtime/src/test/java/org/apache/flink/runtime/io/network/partition/SortMergeResultPartitionReadSchedulerTest.java index 0d1613e14017e..a3b459f0f8fdb 100644 --- a/flink-runtime/src/test/java/org/apache/flink/runtime/io/network/partition/SortMergeResultPartitionReadSchedulerTest.java +++ b/flink-runtime/src/test/java/org/apache/flink/runtime/io/network/partition/SortMergeResultPartitionReadSchedulerTest.java @@ -266,16 +266,16 @@ void testRequestBufferTimeout() throws Exception { SortMergeResultPartitionReadScheduler readScheduler = new SortMergeResultPartitionReadScheduler( bufferPool, executorService, this, bufferRequestTimeout); + long startTimestamp = System.nanoTime(); readScheduler.createSubpartitionReader( new NoOpBufferAvailablityListener(), 0, partitionedFile); // request and use all buffers of buffer pool. readScheduler.run(); assertThat(bufferPool.getAvailableBuffers()).isZero(); - long startTimestamp = System.nanoTime(); assertThatThrownBy(readScheduler::allocateBuffers).isInstanceOf(TimeoutException.class); long requestDuration = System.nanoTime() - startTimestamp; - assertThat(requestDuration > bufferRequestTimeout.toNanos()).isTrue(); + assertThat(requestDuration >= bufferRequestTimeout.toNanos()).isTrue(); readScheduler.release(); }