tillrohrmann commented on a change in pull request #15741:
URL: https://github.com/apache/flink/pull/15741#discussion_r625038879



##########
File path: 
flink-runtime/src/test/java/org/apache/flink/runtime/rpc/RpcEndpointTest.java
##########
@@ -340,108 +319,54 @@ public void testExecute() throws InterruptedException, 
ExecutionException, Timeo
         }
     }
 
-    /** Tests scheduling runnable with delay specified in number and TimeUnit. 
*/
     @Test
-    public void testScheduleRunnable()
-            throws InterruptedException, ExecutionException, TimeoutException {
-        final Time expectedDelay1 = Time.seconds(1);
-        final Time expectedDelay2 = Time.milliseconds(500);
-        final CompletableFuture<Long> actualDelayMsFuture1 = new 
CompletableFuture<>();
-        final CompletableFuture<Long> actualDelayMsFuture2 = new 
CompletableFuture<>();
-        final RpcEndpoint endpoint = new BaseEndpoint(rpcService);
-        try {
-            endpoint.start();
-            final long startTime = System.currentTimeMillis();
-            endpoint.getMainThreadExecutor()
-                    .schedule(
-                            () -> {
-                                endpoint.validateRunsInMainThread();
-                                actualDelayMsFuture1.complete(
-                                        System.currentTimeMillis() - 
startTime);
-                            },
-                            expectedDelay1.getSize(),
-                            expectedDelay1.getUnit());
-            endpoint.getMainThreadExecutor()
-                    .schedule(
-                            () -> {
-                                endpoint.validateRunsInMainThread();
-                                actualDelayMsFuture2.complete(
-                                        System.currentTimeMillis() - 
startTime);
-                            },
-                            expectedDelay2.getSize(),
-                            expectedDelay2.getUnit());
-            final long actualDelayMs1 =
-                    actualDelayMsFuture1.get(
-                            expectedDelay1.getSize() * 2, 
expectedDelay1.getUnit());
-            final long actualDelayMs2 =
-                    actualDelayMsFuture2.get(
-                            expectedDelay2.getSize() * 2, 
expectedDelay2.getUnit());
-            assertTrue(actualDelayMs1 > expectedDelay1.toMilliseconds() * 0.8);
-            assertTrue(actualDelayMs1 < expectedDelay1.toMilliseconds() * 1.2);
-            assertTrue(actualDelayMs2 > expectedDelay2.toMilliseconds() * 0.8);
-            assertTrue(actualDelayMs2 < expectedDelay2.toMilliseconds() * 1.2);
-        } finally {
-            RpcUtils.terminateRpcEndpoint(endpoint, TIMEOUT);
-        }
+    public void testScheduleRunnableWithDelayInMilliseconds() throws Exception 
{
+        testScheduleWithDelay(
+                (mainThreadExecutor, expectedDelay) ->
+                        mainThreadExecutor.schedule(
+                                () -> {}, expectedDelay.toMillis(), 
TimeUnit.MILLISECONDS));
     }
 
-    /** Tests scheduling callable with delay specified in number and TimeUnit. 
*/
     @Test
-    public void testScheduleCallable()
-            throws InterruptedException, ExecutionException, TimeoutException {
-        final Time expectedDelay1 = Time.seconds(1);
-        final Time expectedDelay2 = Time.milliseconds(500);
-        final CompletableFuture<Long> actualDelayMsFuture1 = new 
CompletableFuture<>();
-        final CompletableFuture<Long> actualDelayMsFuture2 = new 
CompletableFuture<>();
-        final RpcEndpoint endpoint = new BaseEndpoint(rpcService);
-        final int expectedInt = 12345;
-        final String expectedString = "Flink";
-        try {
-            endpoint.start();
-            final long startTime = System.currentTimeMillis();
-            final ScheduledFuture<Integer> intScheduleFuture =
-                    endpoint.getMainThreadExecutor()
-                            .schedule(
-                                    () -> {
-                                        endpoint.validateRunsInMainThread();
-                                        actualDelayMsFuture1.complete(
-                                                System.currentTimeMillis() - 
startTime);
-                                        return expectedInt;
-                                    },
-                                    expectedDelay1.getSize(),
-                                    expectedDelay1.getUnit());
-            final ScheduledFuture<String> stringScheduledFuture =
-                    endpoint.getMainThreadExecutor()
-                            .schedule(
-                                    () -> {
-                                        endpoint.validateRunsInMainThread();
-                                        actualDelayMsFuture2.complete(
-                                                System.currentTimeMillis() - 
startTime);
-                                        return expectedString;
-                                    },
-                                    expectedDelay2.getSize(),
-                                    expectedDelay2.getUnit());
-
-            final long actualDelayMs1 =
-                    actualDelayMsFuture1.get(
-                            expectedDelay1.getSize() * 2, 
expectedDelay1.getUnit());
-            final long actualDelayMs2 =
-                    actualDelayMsFuture2.get(
-                            expectedDelay2.getSize() * 2, 
expectedDelay2.getUnit());
-            final int actualInteger =
-                    intScheduleFuture.get(expectedDelay1.getSize() * 2, 
expectedDelay1.getUnit());
-            final String actualString =
-                    stringScheduledFuture.get(
-                            expectedDelay2.getSize() * 2, 
expectedDelay2.getUnit());
-            assertTrue(actualDelayMs1 > expectedDelay1.toMilliseconds() * 0.8);
-            assertTrue(actualDelayMs1 < expectedDelay1.toMilliseconds() * 1.2);
-            assertTrue(actualDelayMs2 > expectedDelay2.toMilliseconds() * 0.8);
-            assertTrue(actualDelayMs2 < expectedDelay2.toMilliseconds() * 1.2);
-            assertEquals(expectedInt, actualInteger);
-            assertEquals(expectedString, actualString);
-        } finally {
-            RpcUtils.terminateRpcEndpoint(endpoint, TIMEOUT);
-        }
+    public void testScheduleRunnableWithDelayInSeconds() throws Exception {

Review comment:
       Ah ok, so we are testing the conversion in 
`RpcEndpoint.MainThreadExecutor` here. This makes sense then.




-- 
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.

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org


Reply via email to