This is an automated email from the ASF dual-hosted git repository.
altay pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/beam.git
The following commit(s) were added to refs/heads/master by this push:
new 9a6f7699b5d [BEAM-14170] - Create a test that runs sickbayed tests
(#17471)
9a6f7699b5d is described below
commit 9a6f7699b5d8daf846221d522d3702c5a4c7b562
Author: Fernando Morales <[email protected]>
AuthorDate: Sun May 29 14:06:52 2022 -0600
[BEAM-14170] - Create a test that runs sickbayed tests (#17471)
---
.../jenkins/job_PostCommit_Java_Sickbay.groovy | 43 ++++++++++++
build.gradle.kts | 8 +++
runners/direct-java/build.gradle | 38 ++++++++++-
runners/flink/flink_runner.gradle | 35 ++++++++--
runners/portability/java/build.gradle | 29 +++++++-
runners/samza/build.gradle | 78 ++++++++++++++--------
runners/spark/job-server/spark_job_server.gradle | 37 ++++++++--
.../sdk/io/BoundedReadFromUnboundedSourceTest.java | 2 -
.../org/apache/beam/sdk/transforms/WatchTest.java | 2 -
9 files changed, 225 insertions(+), 47 deletions(-)
diff --git a/.test-infra/jenkins/job_PostCommit_Java_Sickbay.groovy
b/.test-infra/jenkins/job_PostCommit_Java_Sickbay.groovy
new file mode 100644
index 00000000000..6d2a97fb4f9
--- /dev/null
+++ b/.test-infra/jenkins/job_PostCommit_Java_Sickbay.groovy
@@ -0,0 +1,43 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+import CommonJobProperties as commonJobProperties
+import PostcommitJobBuilder
+
+// This job runs the Java sickbay tests.
+PostcommitJobBuilder.postCommitJob('beam_PostCommit_Java_Sickbay',
+ 'Run Java Sickbay', 'Java Sickbay Tests', this) {
+
+ description('Run Java Sickbay Tests')
+
+ // Set common parameters.
+ commonJobProperties.setTopLevelMainJobProperties(delegate, 'master', 120)
+
+ publishers {
+ archiveJunit('**/build/test-results/**/*.xml')
+ }
+
+ // Execute shell command to run sickbay tests.
+ steps {
+ gradle {
+ rootBuildScriptDir(commonJobProperties.checkoutDir)
+ tasks(':javaPostCommitSickbay')
+ commonJobProperties.setGradleSwitches(delegate)
+ }
+ }
+ }
diff --git a/build.gradle.kts b/build.gradle.kts
index ef0dc431bc8..fd8d8714177 100644
--- a/build.gradle.kts
+++ b/build.gradle.kts
@@ -207,6 +207,14 @@ tasks.register("javaPostCommit") {
dependsOn(":sdks:java:io:neo4j:integrationTest")
}
+tasks.register("javaPostCommitSickbay") {
+ dependsOn(":runners:samza:validatesRunnerSickbay")
+ dependsOn(":runners:flink:validatesRunnerSickbay")
+ dependsOn(":runners:spark:validatesRunnerSickbay")
+ dependsOn(":runners:direct-java:validatesRunnerSickbay")
+ dependsOn(":runners:portability:java:validatesRunnerSickbay")
+}
+
tasks.register("javaHadoopVersionsTest") {
dependsOn(":sdks:java:io:hadoop-common:hadoopVersionsTest")
dependsOn(":sdks:java:io:hadoop-file-system:hadoopVersionsTest")
diff --git a/runners/direct-java/build.gradle b/runners/direct-java/build.gradle
index 3e6c97945a5..995d1f2fd1f 100644
--- a/runners/direct-java/build.gradle
+++ b/runners/direct-java/build.gradle
@@ -115,6 +115,15 @@ static def
pipelineOptionsStringCrossPlatformHandling(String[] options) {
}
}
+def sickbayTests = [
+ // https://issues.apache.org/jira/browse/BEAM-2791
+ 'org.apache.beam.sdk.testing.UsesLoopingTimer',
+ // https://issues.apache.org/jira/browse/BEAM-8035
+
'org.apache.beam.sdk.transforms.WatchTest.testMultiplePollsWithManyResults',
+ // https://issues.apache.org/jira/browse/BEAM-6354
+
'org.apache.beam.sdk.io.BoundedReadFromUnboundedSourceTest.testTimeBound',
+]
+
task needsRunnerTests(type: Test) {
group = "Verification"
description = "Runs tests that require a runner to validate that
piplines/transforms work correctly"
@@ -131,6 +140,13 @@ task needsRunnerTests(type: Test) {
testClassesDirs +=
files(project(":runners:core-java").sourceSets.test.output.classesDirs)
testClassesDirs +=
files(project(":sdks:java:core").sourceSets.test.output.classesDirs)
useJUnit {
+
+ filter {
+ for (String test : sickbayTests) {
+ excludeTestsMatching test
+ }
+ }
+
includeCategories "org.apache.beam.sdk.testing.NeedsRunner"
// Should be run only in a properly configured SDK harness environment
excludeCategories 'org.apache.beam.sdk.testing.UsesSdkHarnessEnvironment'
@@ -140,8 +156,6 @@ task needsRunnerTests(type: Test) {
excludeCategories "org.apache.beam.sdk.testing.UsesJavaExpansionService"
excludeCategories "org.apache.beam.sdk.testing.UsesPythonExpansionService"
excludeCategories 'org.apache.beam.sdk.testing.UsesBundleFinalizer'
- // https://issues.apache.org/jira/browse/BEAM-2791
- excludeCategories 'org.apache.beam.sdk.testing.UsesLoopingTimer'
}
testLogging {
outputs.upToDateWhen {false}
@@ -178,6 +192,26 @@ task validatesRunner(type: Test) {
}
}
+tasks.register("validatesRunnerSickbay", Test) {
+ group = "Verification"
+ description "Validates Direct runner (Sickbay Tests)"
+ systemProperty "beamTestPipelineOptions", JsonOutput.toJson([
+ "--runner=DirectRunner",
+ ])
+
+ classpath = configurations.needsRunner
+ testClassesDirs =
files(project(":sdks:java:core").sourceSets.test.output.classesDirs)
+
+ filter {
+ for (String test : sickbayTests) {
+ includeTestsMatching test
+ }
+
+ // https://issues.apache.org/jira/browse/BEAM-4191
+ includeTestsMatching
'org.apache.beam.runners.direct.WatermarkManagerTest.updateWatermarkWithDifferentWindowedValueInstances'
+ }
+}
+
def gcpProject = project.findProperty('gcpProject') ?: 'apache-beam-testing'
def gcsBucket = project.findProperty('gcsBucket') ?:
'temp-storage-for-release-validation-tests/nightly-snapshot-validation'
def bqDataset = project.findProperty('bqDataset') ?:
'beam_postrelease_mobile_gaming'
diff --git a/runners/flink/flink_runner.gradle
b/runners/flink/flink_runner.gradle
index 18cf7860ff3..156e1a12e82 100644
--- a/runners/flink/flink_runner.gradle
+++ b/runners/flink/flink_runner.gradle
@@ -220,8 +220,14 @@ class ValidatesRunnerConfig {
String name
boolean streaming
boolean checkpointing
+ ArrayList<String> sickbayTests
}
+def sickbayTests = [
+ // TODO(BEAM-13573)
+
'org.apache.beam.sdk.transforms.ParDoTest$TimestampTests.testOnWindowTimestampSkew',
+]
+
def createValidatesRunnerTask(Map m) {
def config = m as ValidatesRunnerConfig
tasks.register(config.name, Test) {
@@ -279,27 +285,29 @@ def createValidatesRunnerTask(Map m) {
excludeCategories 'org.apache.beam.sdk.testing.UsesPerKeyOrderInBundle'
}
filter {
+ for (String test : config.sickbayTests) {
+ excludeTestsMatching test
+ }
+
// https://issues.apache.org/jira/browse/BEAM-12039
excludeTestsMatching
'org.apache.beam.sdk.testing.TestStreamTest.testDiscardingMode'
// https://issues.apache.org/jira/browse/BEAM-12037
excludeTestsMatching
'org.apache.beam.sdk.testing.TestStreamTest.testFirstElementLate'
// https://issues.apache.org/jira/browse/BEAM-12038
excludeTestsMatching
'org.apache.beam.sdk.testing.TestStreamTest.testLateDataAccumulating'
- // TODO(BEAM-13573)
- excludeTestsMatching
'org.apache.beam.sdk.transforms.ParDoTest$TimestampTests.testOnWindowTimestampSkew'
}
}
}
}
-createValidatesRunnerTask(name: "validatesRunnerBatch", streaming: false)
-createValidatesRunnerTask(name: "validatesRunnerStreaming", streaming: true)
+createValidatesRunnerTask(name: "validatesRunnerBatch", streaming: false,
sickbayTests: sickbayTests)
+createValidatesRunnerTask(name: "validatesRunnerStreaming", streaming: true,
sickbayTests: sickbayTests)
// We specifically have a variant which runs with checkpointing enabled for the
// tests that require it since running a checkpoint variant is significantly
// slower since we have to wait shutdownSourcesAfterIdleMs before the source
// can shutdown because of https://issues.apache.org/jira/browse/FLINK-2491
// not supporting checkpointing when an operator has been shutdown.
-createValidatesRunnerTask(name: "validatesRunnerStreamingCheckpointing",
streaming: true, checkpointing: true)
+createValidatesRunnerTask(name: "validatesRunnerStreamingCheckpointing",
streaming: true, checkpointing: true, sickbayTests: sickbayTests)
tasks.register('validatesRunner') {
group = 'Verification'
@@ -309,6 +317,23 @@ tasks.register('validatesRunner') {
dependsOn validatesRunnerStreamingCheckpointing
}
+tasks.register("validatesRunnerSickbay", Test) {
+ group = "Verification"
+ description "Validates Flink runner (Sickbay Tests)"
+ systemProperty "beamTestPipelineOptions", JsonOutput.toJson([
+ "--runner=TestFlinkRunner",
+ ])
+
+ classpath = configurations.validatesRunner
+ testClassesDirs =
files(project(":sdks:java:core").sourceSets.test.output.classesDirs)
+
+ filter {
+ for (String test : sickbayTests) {
+ includeTestsMatching test
+ }
+ }
+}
+
// Generates :runners:flink:1.13:runQuickstartJavaFlinkLocal
createJavaExamplesArchetypeValidationTask(type: 'Quickstart', runner:
'FlinkLocal')
diff --git a/runners/portability/java/build.gradle
b/runners/portability/java/build.gradle
index f9f25526dea..d78315ee5f0 100644
--- a/runners/portability/java/build.gradle
+++ b/runners/portability/java/build.gradle
@@ -120,6 +120,12 @@ task stopLocalJobService {
startLocalJobService.configure{finalizedBy stopLocalJobService}
+def sickbayTests = [
+
+ //TODO: https://issues.apache.org/jira/browse/BEAM-10995
+
'org.apache.beam.sdk.transforms.windowing.WindowingTest.testWindowPreservation',
+]
+
/**
* Runs Java ValidatesRunner tests against the Universal Local Runner (ULR)
aka local_job_service_main
* with the specified environment type.
@@ -193,8 +199,10 @@ def createUlrValidatesRunnerTask = { name,
environmentType, dockerImageTask = ""
excludeTestsMatching
'org.apache.beam.sdk.transforms.ViewTest.testWindowedMapAsEntrySetSideInput'
excludeTestsMatching
'org.apache.beam.sdk.transforms.ViewTest.testMultimapAsEntrySetSideInput'
- // https://issues.apache.org/jira/browse/BEAM-10995
- excludeTestsMatching
'org.apache.beam.sdk.transforms.windowing.WindowingTest.testWindowPreservation'
+ for (String test : sickbayTests) {
+ excludeTestsMatching test
+ }
+
}
}
@@ -213,6 +221,23 @@ def createUlrValidatesRunnerTask = { name,
environmentType, dockerImageTask = ""
return vrTask
}
+tasks.register("validatesRunnerSickbay", Test) {
+ group = "Verification"
+ description "Validates Universal local runner (Sickbay Tests)"
+ systemProperty "beamTestPipelineOptions", JsonOutput.toJson([
+ "--runner=TestUniversalRunner",
+ ])
+
+ classpath = configurations.validatesRunner
+ testClassesDirs =
files(project(":sdks:java:core").sourceSets.test.output.classesDirs)
+
+ filter {
+ for (String test : sickbayTests) {
+ includeTestsMatching test
+ }
+ }
+}
+
task ulrDockerValidatesRunner {
dependsOn createUlrValidatesRunnerTask("ulrDockerValidatesRunnerTests",
"DOCKER", ":sdks:java:container:java8:docker")
}
diff --git a/runners/samza/build.gradle b/runners/samza/build.gradle
index 68ea1780bd4..04a05c18787 100644
--- a/runners/samza/build.gradle
+++ b/runners/samza/build.gradle
@@ -20,11 +20,11 @@ import groovy.json.JsonOutput
plugins { id 'org.apache.beam.module' }
applyJavaNature(
- exportJavadoc: false,
- classesTriggerCheckerBugs: [
- 'GroupWithoutRepartition':
'https://github.com/typetools/checker-framework/issues/3791',
- ],
- automaticModuleName: 'org.apache.beam.runners.samza',
+ exportJavadoc: false,
+ classesTriggerCheckerBugs: [
+ 'GroupWithoutRepartition':
'https://github.com/typetools/checker-framework/issues/3791',
+ ],
+ automaticModuleName: 'org.apache.beam.runners.samza',
)
description = "Apache Beam :: Runners :: Samza"
@@ -86,6 +86,31 @@ configurations.all {
exclude group: "org.slf4j", module: "slf4j-jdk14"
}
+def sickbayTests = [
+ // TODO(BEAM-12750)
+
'org.apache.beam.sdk.transforms.GroupIntoBatchesTest.testInGlobalWindowBatchSizeByteSizeFn',
+
'org.apache.beam.sdk.transforms.GroupIntoBatchesTest.testInStreamingMode',
+
'org.apache.beam.sdk.transforms.GroupIntoBatchesTest.testWithShardedKeyInGlobalWindow',
+ // TODO(BEAM-12749)
+ 'org.apache.beam.sdk.transforms.MapElementsTest.testMapSimpleFunction',
+ // TODO(BEAM-12748)
+ 'org.apache.beam.sdk.transforms.ViewTest.testEmptySingletonSideInput',
+ 'org.apache.beam.sdk.transforms.ViewTest.testNonSingletonSideInput',
+ // TODO(BEAM-12747)
+
'org.apache.beam.sdk.transforms.WithTimestampsTest.withTimestampsBackwardsInTimeShouldThrow',
+
'org.apache.beam.sdk.transforms.WithTimestampsTest.withTimestampsWithNullTimestampShouldThrow',
+ // TODO(BEAM-12746)
+ 'org.apache.beam.sdk.io.FileIOTest*',
+ // TODO(BEAM-12745)
+ 'org.apache.beam.sdk.io.AvroIOTest*',
+ // TODO(BEAM-12744)
+ 'org.apache.beam.sdk.PipelineTest.testEmptyPipeline',
+ // TODO(BEAM-12743)
+
'org.apache.beam.sdk.coders.PCollectionCustomCoderTest.testEncodingNPException',
+
'org.apache.beam.sdk.coders.PCollectionCustomCoderTest.testEncodingIOException',
+
'org.apache.beam.sdk.coders.PCollectionCustomCoderTest.testDecodingNPException',
+
'org.apache.beam.sdk.coders.PCollectionCustomCoderTest.testDecodingIOException',
+]
tasks.register("validatesRunner", Test) {
group = "Verification"
description "Validates Samza runner"
@@ -116,6 +141,9 @@ tasks.register("validatesRunner", Test) {
excludeCategories 'org.apache.beam.sdk.testing.UsesLoopingTimer'
}
filter {
+ for (String test : sickbayTests) {
+ excludeTestsMatching test
+ }
// TODO(BEAM-10025)
excludeTestsMatching
'org.apache.beam.sdk.transforms.ParDoTest$TimerTests.testOutputTimestampDefaultUnbounded'
// TODO(BEAM-11479)
@@ -126,29 +154,6 @@ tasks.register("validatesRunner", Test) {
excludeTestsMatching
'org.apache.beam.sdk.testing.TestStreamTest.testFirstElementLate'
// TODO(BEAM-12036)
excludeTestsMatching
'org.apache.beam.sdk.testing.TestStreamTest.testLateDataAccumulating'
- // TODO(BEAM-12743)
- excludeTestsMatching
'org.apache.beam.sdk.coders.PCollectionCustomCoderTest.testEncodingNPException'
- excludeTestsMatching
'org.apache.beam.sdk.coders.PCollectionCustomCoderTest.testEncodingIOException'
- excludeTestsMatching
'org.apache.beam.sdk.coders.PCollectionCustomCoderTest.testDecodingNPException'
- excludeTestsMatching
'org.apache.beam.sdk.coders.PCollectionCustomCoderTest.testDecodingIOException'
- // TODO(BEAM-12744)
- excludeTestsMatching 'org.apache.beam.sdk.PipelineTest.testEmptyPipeline'
- // TODO(BEAM-12745)
- excludeTestsMatching 'org.apache.beam.sdk.io.AvroIOTest*'
- // TODO(BEAM-12746)
- excludeTestsMatching 'org.apache.beam.sdk.io.FileIOTest*'
- // TODO(BEAM-12747)
- excludeTestsMatching
'org.apache.beam.sdk.transforms.WithTimestampsTest.withTimestampsBackwardsInTimeShouldThrow'
- excludeTestsMatching
'org.apache.beam.sdk.transforms.WithTimestampsTest.withTimestampsWithNullTimestampShouldThrow'
- // TODO(BEAM-12748)
- excludeTestsMatching
'org.apache.beam.sdk.transforms.ViewTest.testEmptySingletonSideInput'
- excludeTestsMatching
'org.apache.beam.sdk.transforms.ViewTest.testNonSingletonSideInput'
- // TODO(BEAM-12749)
- excludeTestsMatching
'org.apache.beam.sdk.transforms.MapElementsTest.testMapSimpleFunction'
- // TODO(BEAM-12750)
- excludeTestsMatching
'org.apache.beam.sdk.transforms.GroupIntoBatchesTest.testInGlobalWindowBatchSizeByteSizeFn'
- excludeTestsMatching
'org.apache.beam.sdk.transforms.GroupIntoBatchesTest.testInStreamingMode'
- excludeTestsMatching
'org.apache.beam.sdk.transforms.GroupIntoBatchesTest.testWithShardedKeyInGlobalWindow'
// These tests fail since there is no support for side inputs in Samza's
unbounded splittable DoFn integration
excludeTestsMatching
'org.apache.beam.sdk.transforms.SplittableDoFnTest.testWindowedSideInputWithCheckpointsUnbounded'
@@ -166,5 +171,22 @@ tasks.register("validatesRunner", Test) {
}
}
+tasks.register("validatesRunnerSickbay", Test) {
+ group = "Verification"
+ description "Validates Samza runner (Sickbay Tests)"
+ systemProperty "beamTestPipelineOptions", JsonOutput.toJson([
+ "--runner=TestSamzaRunner",
+ ])
+
+ classpath = configurations.validatesRunner
+ testClassesDirs =
files(project(":sdks:java:core").sourceSets.test.output.classesDirs)
+
+ filter {
+ for (String test : sickbayTests) {
+ includeTestsMatching test
+ }
+ }
+}
+
// Generates :runners:samza:runQuickstartJavaSamza
createJavaExamplesArchetypeValidationTask(type: 'Quickstart', runner:'Samza')
diff --git a/runners/spark/job-server/spark_job_server.gradle
b/runners/spark/job-server/spark_job_server.gradle
index 18ab88f7b0d..ea7df2c3c0f 100644
--- a/runners/spark/job-server/spark_job_server.gradle
+++ b/runners/spark/job-server/spark_job_server.gradle
@@ -1,3 +1,4 @@
+import groovy.json.JsonOutput
import org.apache.beam.gradle.BeamModulePlugin
/*
@@ -83,7 +84,12 @@ runShadow {
jvmArgs +=
["-Dorg.slf4j.simpleLogger.defaultLogLevel=${project.property('logLevel')}"]
}
-def portableValidatesRunnerTask(String name, boolean streaming, boolean
docker) {
+def sickbayTests = [
+ // TODO(BEAM-13498)
+
'org.apache.beam.sdk.transforms.ParDoTest$TimestampTests.testProcessElementSkew',
+]
+
+def portableValidatesRunnerTask(String name, boolean streaming, boolean
docker, ArrayList<String> sickbayTests) {
def pipelineOptions = []
def testCategories
def testFilter
@@ -185,8 +191,10 @@ def portableValidatesRunnerTask(String name, boolean
streaming, boolean docker)
testFilter = {
// TODO (BEAM-10094)
excludeTestsMatching
'org.apache.beam.sdk.transforms.FlattenTest.testFlattenWithDifferentInputAndOutputCoders2'
- // TODO(BEAM-13498)
- excludeTestsMatching
'org.apache.beam.sdk.transforms.ParDoTest$TimestampTests.testProcessElementSkew'
+
+ for (String test : sickbayTests) {
+ excludeTestsMatching test
+ }
}
}
}
@@ -209,9 +217,9 @@ def portableValidatesRunnerTask(String name, boolean
streaming, boolean docker)
)
}
-project.ext.validatesPortableRunnerDocker=
portableValidatesRunnerTask("Docker", false, true)
-project.ext.validatesPortableRunnerBatch =
portableValidatesRunnerTask("Batch", false, false)
-project.ext.validatesPortableRunnerStreaming =
portableValidatesRunnerTask("Streaming", true, false)
+project.ext.validatesPortableRunnerDocker=
portableValidatesRunnerTask("Docker", false, true, sickbayTests)
+project.ext.validatesPortableRunnerBatch =
portableValidatesRunnerTask("Batch", false, false, sickbayTests)
+project.ext.validatesPortableRunnerStreaming =
portableValidatesRunnerTask("Streaming", true, false, sickbayTests)
tasks.register("validatesPortableRunner") {
dependsOn validatesPortableRunnerDocker
@@ -219,6 +227,23 @@ tasks.register("validatesPortableRunner") {
dependsOn validatesPortableRunnerStreaming
}
+tasks.register("validatesRunnerSickbay", Test) {
+ group = "Verification"
+ description "Validates Spark runner (Sickbay Tests)"
+ systemProperty "beamTestPipelineOptions", JsonOutput.toJson([
+ "--runner=TestSparkRunner",
+ ])
+
+ classpath = configurations.validatesPortableRunner
+ testClassesDirs =
files(project(":sdks:java:core").sourceSets.test.output.classesDirs)
+
+ filter {
+ for (String test : sickbayTests) {
+ includeTestsMatching test
+ }
+ }
+}
+
def jobPort = BeamModulePlugin.getRandomPort()
def artifactPort = BeamModulePlugin.getRandomPort()
diff --git
a/sdks/java/core/src/test/java/org/apache/beam/sdk/io/BoundedReadFromUnboundedSourceTest.java
b/sdks/java/core/src/test/java/org/apache/beam/sdk/io/BoundedReadFromUnboundedSourceTest.java
index 887b23b680c..872e62120d2 100644
---
a/sdks/java/core/src/test/java/org/apache/beam/sdk/io/BoundedReadFromUnboundedSourceTest.java
+++
b/sdks/java/core/src/test/java/org/apache/beam/sdk/io/BoundedReadFromUnboundedSourceTest.java
@@ -46,7 +46,6 @@ import org.apache.beam.sdk.values.PCollection;
import org.checkerframework.checker.nullness.qual.Nullable;
import org.joda.time.Duration;
import org.joda.time.Instant;
-import org.junit.Ignore;
import org.junit.Rule;
import org.junit.Test;
import org.junit.experimental.categories.Category;
@@ -75,7 +74,6 @@ public class BoundedReadFromUnboundedSourceTest implements
Serializable {
}
@Test(timeout = 15000L)
- @Ignore("https://issues.apache.org/jira/browse/BEAM-6354")
@Category(NeedsRunner.class)
public void testTimeBound() throws Exception {
test(false, true);
diff --git
a/sdks/java/core/src/test/java/org/apache/beam/sdk/transforms/WatchTest.java
b/sdks/java/core/src/test/java/org/apache/beam/sdk/transforms/WatchTest.java
index 81873bea303..6aa9d9b58cc 100644
--- a/sdks/java/core/src/test/java/org/apache/beam/sdk/transforms/WatchTest.java
+++ b/sdks/java/core/src/test/java/org/apache/beam/sdk/transforms/WatchTest.java
@@ -78,7 +78,6 @@ import org.checkerframework.checker.nullness.qual.Nullable;
import org.joda.time.Duration;
import org.joda.time.Instant;
import org.joda.time.ReadableDuration;
-import org.junit.Ignore;
import org.junit.Rule;
import org.junit.Test;
import org.junit.experimental.categories.Category;
@@ -313,7 +312,6 @@ public class WatchTest implements Serializable {
@Test
@Category({NeedsRunner.class, UsesUnboundedSplittableParDo.class})
- @Ignore("https://issues.apache.org/jira/browse/BEAM-8035")
public void testMultiplePollsWithManyResults() {
final long numResults = 3000;
List<Integer> all = Lists.newArrayList();