This is an automated email from the ASF dual-hosted git repository.

dwysakowicz pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/flink-benchmarks.git


The following commit(s) were added to refs/heads/master by this push:
     new f40b69e  [FLINK-23284] Reset the TARGET_COUNT_REACHED_LATCH before 
each execution of the job
f40b69e is described below

commit f40b69e6341118720ff941b91e98729baf7e0224
Author: Dawid Wysakowicz <[email protected]>
AuthorDate: Tue Jul 6 16:52:57 2021 +0200

    [FLINK-23284] Reset the TARGET_COUNT_REACHED_LATCH before each execution of 
the job
    
    The Level.Iteration is not enough to clear up the LATCH before every run of 
the readFileSplit method. There might be multiple calls to the benchmarked 
method within a single iteration. If it is the case, the second invocation did 
not have the latch properly reset, which resulted in all records being emitted 
in the finishing stage of the ContinuousFileReadingOperator. This led to 
testing a different behaviour than the envisioned.
---
 .../flink/benchmark/ContinuousFileReaderOperatorBenchmark.java      | 6 +-----
 1 file changed, 1 insertion(+), 5 deletions(-)

diff --git 
a/src/main/java/org/apache/flink/benchmark/ContinuousFileReaderOperatorBenchmark.java
 
b/src/main/java/org/apache/flink/benchmark/ContinuousFileReaderOperatorBenchmark.java
index b4937c7..524bf5d 100644
--- 
a/src/main/java/org/apache/flink/benchmark/ContinuousFileReaderOperatorBenchmark.java
+++ 
b/src/main/java/org/apache/flink/benchmark/ContinuousFileReaderOperatorBenchmark.java
@@ -66,13 +66,9 @@ public class ContinuousFileReaderOperatorBenchmark extends 
BenchmarkBase {
         new Runner(options).run();
     }
 
-    @TearDown(Level.Iteration)
-    public void tearDown() {
-        TARGET_COUNT_REACHED_LATCH.reset();
-    }
-
     @Benchmark
     public void readFileSplit(FlinkEnvironmentContext context) throws 
Exception {
+        TARGET_COUNT_REACHED_LATCH.reset();
         StreamExecutionEnvironment env = context.env;
         env.setRestartStrategy(new 
RestartStrategies.NoRestartStrategyConfiguration());
         env

Reply via email to