From 7833ec09bf9ff49183662965f3ff05f8cbc6ac54 Mon Sep 17 00:00:00 2001 From: Jason Kuster Date: Fri, 29 Apr 2016 17:25:23 -0700 Subject: [PATCH 1/8] Add Matcher serializer in TestPipeline. --- .../org/apache/beam/examples/WordCountIT.java | 5 - .../testing/TestDataflowPipelineRunner.java | 15 +- .../TestDataflowPipelineRunnerTest.java | 251 ++++++++++++++++++ .../apache/beam/sdk/testing/TestPipeline.java | 94 +++++++ .../beam/sdk/testing/TestPipelineOptions.java | 23 ++ .../beam/sdk/testing/TestPipelineTest.java | 4 + 6 files changed, 380 insertions(+), 12 deletions(-) diff --git a/examples/java/src/test/java/org/apache/beam/examples/WordCountIT.java b/examples/java/src/test/java/org/apache/beam/examples/WordCountIT.java index 56ca98c1a6b7..7c7acb0bd1ee 100644 --- a/examples/java/src/test/java/org/apache/beam/examples/WordCountIT.java +++ b/examples/java/src/test/java/org/apache/beam/examples/WordCountIT.java @@ -55,10 +55,5 @@ public void testE2EWordCount() throws Exception { options.getJobName(), "output", "results"})); WordCount.main(TestPipeline.convertToArgs(options)); - PipelineResult result = - TestDataflowPipelineRunner.getPipelineResultByJobName(options.getJobName()); - - assertNotNull("Result was null.", result); - assertEquals("Pipeline state was not done.", PipelineResult.State.DONE, result.getState()); } } diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunner.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunner.java index 3ab91f5511df..17e31e892094 100644 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunner.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunner.java @@ -17,6 +17,8 @@ */ package org.apache.beam.runners.dataflow.testing; +import static org.hamcrest.MatcherAssert.assertThat; + import org.apache.beam.runners.dataflow.DataflowJobExecutionException; import org.apache.beam.runners.dataflow.DataflowPipelineJob; import org.apache.beam.runners.dataflow.DataflowPipelineRunner; @@ -40,6 +42,7 @@ import com.google.common.base.Optional; import com.google.common.base.Throwables; +import org.hamcrest.Matcher; import org.slf4j.Logger; import org.slf4j.LoggerFactory; @@ -62,8 +65,6 @@ public class TestDataflowPipelineRunner extends PipelineRunner { private static final String TENTATIVE_COUNTER = "tentative"; private static final Logger LOG = LoggerFactory.getLogger(TestDataflowPipelineRunner.class); - private static final Map EXECUTION_RESULTS = - new ConcurrentHashMap(); private final TestDataflowPipelineOptions options; private final DataflowPipelineRunner runner; @@ -87,10 +88,6 @@ public static TestDataflowPipelineRunner fromOptions( return new TestDataflowPipelineRunner(dataflowOptions); } - public static PipelineResult getPipelineResultByJobName(String jobName) { - return EXECUTION_RESULTS.get(jobName); - } - @Override public DataflowPipelineJob run(Pipeline pipeline) { return run(pipeline, runner); @@ -98,6 +95,7 @@ public DataflowPipelineJob run(Pipeline pipeline) { DataflowPipelineJob run(Pipeline pipeline, DataflowPipelineRunner runner) { + TestPipelineOptions testPipelineOptions = pipeline.getOptions().as(TestPipelineOptions.class); final DataflowPipelineJob job; try { job = runner.run(pipeline); @@ -108,6 +106,8 @@ DataflowPipelineJob run(Pipeline pipeline, DataflowPipelineRunner runner) { LOG.info("Running Dataflow job {} with {} expected assertions.", job.getJobId(), expectedNumberOfAssertions); + assertThat(job, testPipelineOptions.getOnCreateMatcher()); + CancelWorkflowOnError messageHandler = new CancelWorkflowOnError( job, new MonitoringUtil.PrintHandler(options.getJobMessageOutput())); @@ -151,6 +151,8 @@ public Optional call() throws Exception { throw new AssertionError(messageHandler.getErrorMessage() == null ? "The dataflow did not return a failure reason." : messageHandler.getErrorMessage()); + } else { + assertThat(job, testPipelineOptions.getOnSuccessMatcher()); } } catch (InterruptedException e) { Thread.currentThread().interrupt(); @@ -161,7 +163,6 @@ public Optional call() throws Exception { } catch (IOException e) { throw new RuntimeException(e); } - EXECUTION_RESULTS.put(options.getJobName(), job); return job; } diff --git a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunnerTest.java b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunnerTest.java index a45284c9afa9..1cdd158bb656 100644 --- a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunnerTest.java +++ b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunnerTest.java @@ -26,6 +26,7 @@ import static org.mockito.Mockito.atLeastOnce; import static org.mockito.Mockito.doCallRealMethod; import static org.mockito.Mockito.doReturn; +import static org.mockito.Mockito.never; import static org.mockito.Mockito.spy; import static org.mockito.Mockito.verify; import static org.mockito.Mockito.when; @@ -36,6 +37,7 @@ import org.apache.beam.runners.dataflow.util.MonitoringUtil.JobMessagesHandler; import org.apache.beam.runners.dataflow.util.TimeUtil; import org.apache.beam.sdk.Pipeline; +import org.apache.beam.sdk.PipelineResult; import org.apache.beam.sdk.PipelineResult.State; import org.apache.beam.sdk.options.PipelineOptionsFactory; import org.apache.beam.sdk.testing.PAssert; @@ -61,6 +63,8 @@ import com.google.common.collect.ImmutableMap; import com.google.common.collect.Lists; +import org.hamcrest.BaseMatcher; +import org.hamcrest.Description; import org.joda.time.Instant; import org.junit.Before; import org.junit.Rule; @@ -378,4 +382,251 @@ public State answer(InvocationOnMock invocation) { // instead of inside the try-catch block. fail("AssertionError expected"); } + + @Test + public void testBatchCreateMatcher() throws Exception { + Pipeline p = TestPipeline.create(options); + PCollection pc = p.apply(Create.of(1, 2, 3)); + PAssert.that(pc).containsInAnyOrder(1, 2, 3); + final DataflowPipelineJob mockJob = Mockito.mock(DataflowPipelineJob.class); + when(mockJob.getDataflowClient()).thenReturn(service); + when(mockJob.getState()).thenReturn(State.DONE); + when(mockJob.getProjectId()).thenReturn("test-project"); + when(mockJob.getJobId()).thenReturn("test-job"); + + DataflowPipelineRunner mockRunner = Mockito.mock(DataflowPipelineRunner.class); + when(mockRunner.run(any(Pipeline.class))).thenReturn(mockJob); + + TestDataflowPipelineRunner runner = (TestDataflowPipelineRunner) p.getRunner(); + p.getOptions().as(TestPipelineOptions.class) + .setOnCreateMatcher(new BaseMatcher() { + @Override + public boolean matches(Object o) { + try { + verify(mockJob, never()).waitToFinish(any(Long.class), any(TimeUnit.class), + any(JobMessagesHandler.class)); + } catch (Exception e) { + return false; + } + return (PipelineResult) o == (PipelineResult) mockJob; + } + + @Override + public void describeTo(Description description) { + } + }); + + when(request.execute()).thenReturn( + generateMockMetricResponse(true /* success */, true /* tentative */)); + runner.run(p, mockRunner); + } + + @Test + public void testStreamingCreateMatcher() throws Exception { + options.setStreaming(true); + Pipeline p = TestPipeline.create(options); + PCollection pc = p.apply(Create.of(1, 2, 3)); + PAssert.that(pc).containsInAnyOrder(1, 2, 3); + + final DataflowPipelineJob mockJob = Mockito.mock(DataflowPipelineJob.class); + when(mockJob.getDataflowClient()).thenReturn(service); + when(mockJob.getState()).thenReturn(State.DONE); + when(mockJob.getProjectId()).thenReturn("test-project"); + when(mockJob.getJobId()).thenReturn("test-job"); + + DataflowPipelineRunner mockRunner = Mockito.mock(DataflowPipelineRunner.class); + when(mockRunner.run(any(Pipeline.class))).thenReturn(mockJob); + + TestDataflowPipelineRunner runner = (TestDataflowPipelineRunner) p.getRunner(); + p.getOptions().as(TestPipelineOptions.class) + .setOnCreateMatcher(new BaseMatcher() { + @Override + public boolean matches(Object o) { + // Assert that job.WaitToFinish has not yet been called + try { + verify(mockJob, never()).waitToFinish(any(Long.class), any(TimeUnit.class), + any(JobMessagesHandler.class)); + } catch (Exception e) { + return false; + } + return (PipelineResult) o == (PipelineResult) mockJob; + } + + @Override + public void describeTo(Description description) { + } + }); + when(mockJob.waitToFinish(any(Long.class), any(TimeUnit.class), any(JobMessagesHandler.class))) + .thenReturn(State.DONE); + + when(request.execute()).thenReturn( + generateMockMetricResponse(true /* success */, true /* tentative */)); + runner.run(p, mockRunner); + } + + @Test + public void testBatchSuccessMatcherPipelineSuccess() throws Exception { + Pipeline p = TestPipeline.create(options); + PCollection pc = p.apply(Create.of(1, 2, 3)); + PAssert.that(pc).containsInAnyOrder(1, 2, 3); + + final DataflowPipelineJob mockJob = Mockito.mock(DataflowPipelineJob.class); + when(mockJob.getDataflowClient()).thenReturn(service); + when(mockJob.getState()).thenReturn(State.DONE); + when(mockJob.getProjectId()).thenReturn("test-project"); + when(mockJob.getJobId()).thenReturn("test-job"); + + DataflowPipelineRunner mockRunner = Mockito.mock(DataflowPipelineRunner.class); + when(mockRunner.run(any(Pipeline.class))).thenReturn(mockJob); + + TestDataflowPipelineRunner runner = (TestDataflowPipelineRunner) p.getRunner(); + p.getOptions().as(TestPipelineOptions.class) + .setOnSuccessMatcher(new BaseMatcher() { + @Override + public boolean matches(Object o) { + // Assert that job.WaitToFinish has not yet been called + try { + verify(mockJob, Mockito.times(1)).waitToFinish(any(Long.class), any(TimeUnit.class), + any(JobMessagesHandler.class)); + } catch (Exception e) { + return false; + } + return (PipelineResult) o == (PipelineResult) mockJob; + } + + @Override + public void describeTo(Description description) { + } + }); + + when(request.execute()).thenReturn( + generateMockMetricResponse(true /* success */, true /* tentative */)); + runner.run(p, mockRunner); + } + + @Test + public void testStreamingSuccessMatcherPipelineSuccess() throws Exception { + options.setStreaming(true); + Pipeline p = TestPipeline.create(options); + PCollection pc = p.apply(Create.of(1, 2, 3)); + PAssert.that(pc).containsInAnyOrder(1, 2, 3); + + final DataflowPipelineJob mockJob = Mockito.mock(DataflowPipelineJob.class); + when(mockJob.getDataflowClient()).thenReturn(service); + when(mockJob.getState()).thenReturn(State.DONE); + when(mockJob.getProjectId()).thenReturn("test-project"); + when(mockJob.getJobId()).thenReturn("test-job"); + + DataflowPipelineRunner mockRunner = Mockito.mock(DataflowPipelineRunner.class); + when(mockRunner.run(any(Pipeline.class))).thenReturn(mockJob); + + TestDataflowPipelineRunner runner = (TestDataflowPipelineRunner) p.getRunner(); + p.getOptions().as(TestPipelineOptions.class) + .setOnSuccessMatcher(new BaseMatcher() { + @Override + public boolean matches(Object o) { + // Assert that job.WaitToFinish has not yet been called + try { + verify(mockJob, Mockito.times(1)).waitToFinish(any(Long.class), any(TimeUnit.class), + any(JobMessagesHandler.class)); + } catch (Exception e) { + return false; + } + return (PipelineResult) o == (PipelineResult) mockJob; + } + + @Override + public void describeTo(Description description) { + } + }); + when(mockJob.waitToFinish(any(Long.class), any(TimeUnit.class), any(JobMessagesHandler.class))) + .thenReturn(State.DONE); + + when(request.execute()).thenReturn( + generateMockMetricResponse(true /* success */, true /* tentative */)); + runner.run(p, mockRunner); + } + + @Test + public void testBatchSuccessMatcherPipelineFailure() throws Exception { + Pipeline p = TestPipeline.create(options); + PCollection pc = p.apply(Create.of(1, 2, 3)); + PAssert.that(pc).containsInAnyOrder(1, 2, 3); + + final DataflowPipelineJob mockJob = Mockito.mock(DataflowPipelineJob.class); + when(mockJob.getDataflowClient()).thenReturn(service); + when(mockJob.getState()).thenReturn(State.FAILED); + when(mockJob.getProjectId()).thenReturn("test-project"); + when(mockJob.getJobId()).thenReturn("test-job"); + + DataflowPipelineRunner mockRunner = Mockito.mock(DataflowPipelineRunner.class); + when(mockRunner.run(any(Pipeline.class))).thenReturn(mockJob); + + TestDataflowPipelineRunner runner = (TestDataflowPipelineRunner) p.getRunner(); + p.getOptions().as(TestPipelineOptions.class) + .setOnSuccessMatcher(new BaseMatcher() { + @Override + public boolean matches(Object o) { + throw new AssertionError("Success matcher shouldn't be called."); + } + + @Override + public void describeTo(Description description) { + } + }); + + when(request.execute()).thenReturn( + generateMockMetricResponse(false /* success */, true /* tentative */)); + try { + runner.run(p, mockRunner); + } catch (AssertionError expected) { + verify(mockJob, Mockito.times(1)).waitToFinish(any(Long.class), any(TimeUnit.class), + any(JobMessagesHandler.class)); + return; + } + fail("Expected an exception on pipeline failure."); + } + + @Test + public void testStreamingSuccessMatcherPipelineFailure() throws Exception { + options.setStreaming(true); + Pipeline p = TestPipeline.create(options); + PCollection pc = p.apply(Create.of(1, 2, 3)); + PAssert.that(pc).containsInAnyOrder(1, 2, 3); + + final DataflowPipelineJob mockJob = Mockito.mock(DataflowPipelineJob.class); + when(mockJob.getDataflowClient()).thenReturn(service); + when(mockJob.getState()).thenReturn(State.FAILED); + when(mockJob.getProjectId()).thenReturn("test-project"); + when(mockJob.getJobId()).thenReturn("test-job"); + + DataflowPipelineRunner mockRunner = Mockito.mock(DataflowPipelineRunner.class); + when(mockRunner.run(any(Pipeline.class))).thenReturn(mockJob); + + TestDataflowPipelineRunner runner = (TestDataflowPipelineRunner) p.getRunner(); + p.getOptions().as(TestPipelineOptions.class) + .setOnSuccessMatcher(new BaseMatcher() { + @Override + public boolean matches(Object o) { + throw new AssertionError("Success matcher shouldn't be called."); + } + + @Override + public void describeTo(Description description) { + } + }); + when(mockJob.waitToFinish(any(Long.class), any(TimeUnit.class), any(JobMessagesHandler.class))) + .thenReturn(State.FAILED); + + when(request.execute()).thenReturn( + generateMockMetricResponse(false /* success */, true /* tentative */)); + try { + runner.run(p, mockRunner); + } catch (AssertionError expected) { + verify(mockJob, Mockito.times(1)).waitToFinish(any(Long.class), any(TimeUnit.class), + any(JobMessagesHandler.class)); + return; + } + fail("Expected an exception on pipeline failure."); + } } diff --git a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipeline.java b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipeline.java index a51a24e81270..c903ac7f04ce 100644 --- a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipeline.java +++ b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipeline.java @@ -25,21 +25,37 @@ import org.apache.beam.sdk.options.PipelineOptions.CheckEnabled; import org.apache.beam.sdk.options.PipelineOptionsFactory; import org.apache.beam.sdk.runners.PipelineRunner; +import org.apache.beam.sdk.util.ReleaseInfo; import org.apache.beam.sdk.util.TestCredential; import com.google.common.base.Optional; import com.google.common.base.Strings; import com.google.common.collect.Iterators; +import com.fasterxml.jackson.core.JsonGenerator; +import com.fasterxml.jackson.core.JsonParser; +import com.fasterxml.jackson.core.JsonProcessingException; +import com.fasterxml.jackson.core.Version; +import com.fasterxml.jackson.databind.DeserializationContext; +import com.fasterxml.jackson.databind.JsonDeserializer; +import com.fasterxml.jackson.databind.JsonSerializer; +import com.fasterxml.jackson.databind.Module; import com.fasterxml.jackson.databind.ObjectMapper; +import com.fasterxml.jackson.databind.SerializerProvider; +import com.fasterxml.jackson.databind.module.SimpleDeserializers; +import com.fasterxml.jackson.databind.module.SimpleSerializers; +import org.hamcrest.Matcher; import org.junit.experimental.categories.Category; import java.io.IOException; import java.lang.reflect.Method; import java.util.ArrayList; +import java.util.HashMap; import java.util.Iterator; import java.util.Map; +import java.util.UUID; +import java.util.concurrent.ConcurrentHashMap; import javax.annotation.Nullable; @@ -83,6 +99,10 @@ public class TestPipeline extends Pipeline { private static final String PROPERTY_BEAM_TEST_PIPELINE_OPTIONS = "beamTestPipelineOptions"; private static final ObjectMapper MAPPER = new ObjectMapper(); + static { + MAPPER.registerModule(new IntegrationMatcherStore()); + } + /** * Creates and returns a new test pipeline. * @@ -221,4 +241,78 @@ private static Optional findCallersStackTrace() { } return firstInstanceAfterTestPipeline; } + + private static class IntegrationMatcherStore extends Module { + private static Map> matcherMap = new ConcurrentHashMap<>(); + + /** + * Adds a matcher to the matcher store, to be retrieved by a TestPipelineRunner during + * pipeline execution. + * + * @param m The matcher to add to the store. + * @return Returns a UUID corresponding to the matcher which can be used in e.g. + * opts.SetPassVerifier(matcher) + */ + String addMatcher(Matcher m) { + if (matcherMap == null) { + matcherMap = new HashMap<>(); + } + String uid = UUID.randomUUID().toString(); + matcherMap.put(uid, m); + return uid; + } + + /** + * Retrieves a matcher from the matcher store. + * + * @param str The UUID of the matcher to return. + * @return Returns the matcher corresponding to the string passed, or null if it doesn't exist. + */ + Matcher getMatcher(String str) { + if (matcherMap == null) { + return null; + } + return matcherMap.get(str); + } + + @Override + public String getModuleName() { + return "IntegrationMatcherStore"; + } + + @Override + public Version version() { + return new Version(1, 0, 0, "", null, null); + } + + @Override + public void setupModule(SetupContext setupContext) { + SimpleSerializers ser = new SimpleSerializers(); + ser.addSerializer(Matcher.class, (JsonSerializer) new MatcherSerializer()); + SimpleDeserializers des = new SimpleDeserializers(); + des.addDeserializer(Matcher.class, new MatcherDeserializer()); + setupContext.addSerializers(ser); + setupContext.addDeserializers(des); + } + + class MatcherSerializer extends JsonSerializer> { + @Override + public void serialize(Matcher matcher, JsonGenerator jsonGenerator, + SerializerProvider serializerProvider) throws IOException, JsonProcessingException { + String uuid = addMatcher(matcher); + jsonGenerator.writeString(uuid); + } + } + + class MatcherDeserializer extends JsonDeserializer> { + + @Override + public Matcher deserialize(JsonParser jsonParser, + DeserializationContext deserializationContext) + throws IOException, JsonProcessingException { + String uuid = jsonParser.getText(); + return getMatcher(uuid); + } + } + } } diff --git a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipelineOptions.java b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipelineOptions.java index 2599ae260d9f..5255d5de1878 100644 --- a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipelineOptions.java +++ b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipelineOptions.java @@ -17,7 +17,14 @@ */ package org.apache.beam.sdk.testing; +import org.apache.beam.sdk.PipelineResult; +import org.apache.beam.sdk.options.Default; +import org.apache.beam.sdk.options.DefaultValueFactory; import org.apache.beam.sdk.options.PipelineOptions; +import org.hamcrest.BaseMatcher; +import org.hamcrest.Description; +import org.hamcrest.Matcher; +import org.hamcrest.Matchers; /** * {@link TestPipelineOptions} is a set of options for test pipelines. @@ -27,4 +34,20 @@ public interface TestPipelineOptions extends PipelineOptions { String getTempRoot(); void setTempRoot(String value); + + @Default.InstanceFactory(AlwaysPassMatcherFactory.class) + Matcher getOnCreateMatcher(); + void setOnCreateMatcher(Matcher value); + + @Default.InstanceFactory(AlwaysPassMatcherFactory.class) + Matcher getOnSuccessMatcher(); + void setOnSuccessMatcher(Matcher value); + + public static class AlwaysPassMatcherFactory implements DefaultValueFactory> { + @Override + public Matcher create(PipelineOptions options) { + return Matchers.any(PipelineResult.class); + } + } + } diff --git a/sdks/java/core/src/test/java/org/apache/beam/sdk/testing/TestPipelineTest.java b/sdks/java/core/src/test/java/org/apache/beam/sdk/testing/TestPipelineTest.java index 9460e13d2c84..e42d55ccd563 100644 --- a/sdks/java/core/src/test/java/org/apache/beam/sdk/testing/TestPipelineTest.java +++ b/sdks/java/core/src/test/java/org/apache/beam/sdk/testing/TestPipelineTest.java @@ -23,6 +23,7 @@ import static org.junit.Assert.assertNotNull; import static org.junit.Assert.assertThat; +import org.apache.beam.sdk.PipelineResult; import org.apache.beam.sdk.options.ApplicationNameOptions; import org.apache.beam.sdk.options.GcpOptions; import org.apache.beam.sdk.options.PipelineOptions; @@ -31,6 +32,9 @@ import com.fasterxml.jackson.databind.ObjectMapper; +import org.hamcrest.BaseMatcher; +import org.hamcrest.Description; +import org.hamcrest.Matcher; import org.junit.Rule; import org.junit.Test; import org.junit.rules.TestRule; From 9533e9f9eef13106127d8ce18718901b7666f0ee Mon Sep 17 00:00:00 2001 From: Jason Kuster Date: Tue, 3 May 2016 10:35:52 -0700 Subject: [PATCH 2/8] Checkstyle fixes, etc. --- .../test/java/org/apache/beam/examples/WordCountIT.java | 5 ----- .../dataflow/testing/TestDataflowPipelineRunner.java | 6 +----- .../dataflow/testing/TestDataflowPipelineRunnerTest.java | 1 + .../java/org/apache/beam/sdk/testing/TestPipeline.java | 1 - .../org/apache/beam/sdk/testing/TestPipelineOptions.java | 8 +++++--- .../org/apache/beam/sdk/testing/TestPipelineTest.java | 4 ---- 6 files changed, 7 insertions(+), 18 deletions(-) diff --git a/examples/java/src/test/java/org/apache/beam/examples/WordCountIT.java b/examples/java/src/test/java/org/apache/beam/examples/WordCountIT.java index 7c7acb0bd1ee..4d8c04b55f62 100644 --- a/examples/java/src/test/java/org/apache/beam/examples/WordCountIT.java +++ b/examples/java/src/test/java/org/apache/beam/examples/WordCountIT.java @@ -18,13 +18,8 @@ package org.apache.beam.examples; -import static org.junit.Assert.assertEquals; -import static org.junit.Assert.assertNotNull; - import org.apache.beam.examples.WordCount.WordCountOptions; import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; -import org.apache.beam.runners.dataflow.testing.TestDataflowPipelineRunner; -import org.apache.beam.sdk.PipelineResult; import org.apache.beam.sdk.options.PipelineOptionsFactory; import org.apache.beam.sdk.testing.TestPipeline; import org.apache.beam.sdk.testing.TestPipelineOptions; diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunner.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunner.java index 17e31e892094..2b53a655a0f9 100644 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunner.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunner.java @@ -25,12 +25,12 @@ import org.apache.beam.runners.dataflow.util.MonitoringUtil; import org.apache.beam.runners.dataflow.util.MonitoringUtil.JobMessagesHandler; import org.apache.beam.sdk.Pipeline; -import org.apache.beam.sdk.PipelineResult; import org.apache.beam.sdk.PipelineResult.State; import org.apache.beam.sdk.options.PipelineOptions; import org.apache.beam.sdk.runners.PipelineRunner; import org.apache.beam.sdk.testing.PAssert; import org.apache.beam.sdk.testing.TestPipeline; +import org.apache.beam.sdk.testing.TestPipelineOptions; import org.apache.beam.sdk.transforms.PTransform; import org.apache.beam.sdk.values.PInput; import org.apache.beam.sdk.values.POutput; @@ -41,17 +41,13 @@ import com.google.common.base.Joiner; import com.google.common.base.Optional; import com.google.common.base.Throwables; - -import org.hamcrest.Matcher; import org.slf4j.Logger; import org.slf4j.LoggerFactory; import java.io.IOException; import java.math.BigDecimal; import java.util.List; -import java.util.Map; import java.util.concurrent.Callable; -import java.util.concurrent.ConcurrentHashMap; import java.util.concurrent.ExecutionException; import java.util.concurrent.Future; import java.util.concurrent.TimeUnit; diff --git a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunnerTest.java b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunnerTest.java index 1cdd158bb656..1b0a190aba6b 100644 --- a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunnerTest.java +++ b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunnerTest.java @@ -42,6 +42,7 @@ import org.apache.beam.sdk.options.PipelineOptionsFactory; import org.apache.beam.sdk.testing.PAssert; import org.apache.beam.sdk.testing.TestPipeline; +import org.apache.beam.sdk.testing.TestPipelineOptions; import org.apache.beam.sdk.transforms.Create; import org.apache.beam.sdk.util.GcsUtil; import org.apache.beam.sdk.util.NoopPathValidator; diff --git a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipeline.java b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipeline.java index c903ac7f04ce..7c3a92ccfba8 100644 --- a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipeline.java +++ b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipeline.java @@ -25,7 +25,6 @@ import org.apache.beam.sdk.options.PipelineOptions.CheckEnabled; import org.apache.beam.sdk.options.PipelineOptionsFactory; import org.apache.beam.sdk.runners.PipelineRunner; -import org.apache.beam.sdk.util.ReleaseInfo; import org.apache.beam.sdk.util.TestCredential; import com.google.common.base.Optional; diff --git a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipelineOptions.java b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipelineOptions.java index 5255d5de1878..a15d612ef552 100644 --- a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipelineOptions.java +++ b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipelineOptions.java @@ -21,8 +21,6 @@ import org.apache.beam.sdk.options.Default; import org.apache.beam.sdk.options.DefaultValueFactory; import org.apache.beam.sdk.options.PipelineOptions; -import org.hamcrest.BaseMatcher; -import org.hamcrest.Description; import org.hamcrest.Matcher; import org.hamcrest.Matchers; @@ -43,7 +41,11 @@ public interface TestPipelineOptions extends PipelineOptions { Matcher getOnSuccessMatcher(); void setOnSuccessMatcher(Matcher value); - public static class AlwaysPassMatcherFactory implements DefaultValueFactory> { + /** + * Factory for PipelineResult matchers which always pass. + */ + class AlwaysPassMatcherFactory + implements DefaultValueFactory> { @Override public Matcher create(PipelineOptions options) { return Matchers.any(PipelineResult.class); diff --git a/sdks/java/core/src/test/java/org/apache/beam/sdk/testing/TestPipelineTest.java b/sdks/java/core/src/test/java/org/apache/beam/sdk/testing/TestPipelineTest.java index e42d55ccd563..9460e13d2c84 100644 --- a/sdks/java/core/src/test/java/org/apache/beam/sdk/testing/TestPipelineTest.java +++ b/sdks/java/core/src/test/java/org/apache/beam/sdk/testing/TestPipelineTest.java @@ -23,7 +23,6 @@ import static org.junit.Assert.assertNotNull; import static org.junit.Assert.assertThat; -import org.apache.beam.sdk.PipelineResult; import org.apache.beam.sdk.options.ApplicationNameOptions; import org.apache.beam.sdk.options.GcpOptions; import org.apache.beam.sdk.options.PipelineOptions; @@ -32,9 +31,6 @@ import com.fasterxml.jackson.databind.ObjectMapper; -import org.hamcrest.BaseMatcher; -import org.hamcrest.Description; -import org.hamcrest.Matcher; import org.junit.Rule; import org.junit.Test; import org.junit.rules.TestRule; From afbd4414f71ddb76a87f126d25bba324827d1d0b Mon Sep 17 00:00:00 2001 From: Jason Kuster Date: Tue, 3 May 2016 13:57:25 -0700 Subject: [PATCH 3/8] First wave of pull request fixes. Signed-off-by: Jason Kuster --- .../TestDataflowPipelineRunnerTest.java | 87 ++++++++----------- 1 file changed, 36 insertions(+), 51 deletions(-) diff --git a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunnerTest.java b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunnerTest.java index 1b0a190aba6b..8f43f3cd93a3 100644 --- a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunnerTest.java +++ b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunnerTest.java @@ -19,6 +19,7 @@ import static org.hamcrest.Matchers.containsString; import static org.junit.Assert.assertEquals; +import static org.junit.Assert.assertSame; import static org.junit.Assert.assertThat; import static org.junit.Assert.fail; import static org.mockito.Matchers.any; @@ -65,6 +66,7 @@ import com.google.common.collect.Lists; import org.hamcrest.BaseMatcher; +import org.hamcrest.CustomTypeSafeMatcher; import org.hamcrest.Description; import org.joda.time.Instant; import org.junit.Before; @@ -385,10 +387,11 @@ public State answer(InvocationOnMock invocation) { } @Test - public void testBatchCreateMatcher() throws Exception { + public void testBatchOnCreateMatcher() throws Exception { Pipeline p = TestPipeline.create(options); PCollection pc = p.apply(Create.of(1, 2, 3)); PAssert.that(pc).containsInAnyOrder(1, 2, 3); + final DataflowPipelineJob mockJob = Mockito.mock(DataflowPipelineJob.class); when(mockJob.getDataflowClient()).thenReturn(service); when(mockJob.getState()).thenReturn(State.DONE); @@ -400,20 +403,17 @@ public void testBatchCreateMatcher() throws Exception { TestDataflowPipelineRunner runner = (TestDataflowPipelineRunner) p.getRunner(); p.getOptions().as(TestPipelineOptions.class) - .setOnCreateMatcher(new BaseMatcher() { + .setOnCreateMatcher(new CustomTypeSafeMatcher("TestMatcher") { @Override - public boolean matches(Object o) { + protected boolean matchesSafely(PipelineResult pipelineResult) { try { verify(mockJob, never()).waitToFinish(any(Long.class), any(TimeUnit.class), any(JobMessagesHandler.class)); } catch (Exception e) { return false; } - return (PipelineResult) o == (PipelineResult) mockJob; - } - - @Override - public void describeTo(Description description) { + assertSame(mockJob, pipelineResult); + return true; } }); @@ -423,7 +423,7 @@ public void describeTo(Description description) { } @Test - public void testStreamingCreateMatcher() throws Exception { + public void testStreamingOnCreateMatcher() throws Exception { options.setStreaming(true); Pipeline p = TestPipeline.create(options); PCollection pc = p.apply(Create.of(1, 2, 3)); @@ -440,23 +440,20 @@ public void testStreamingCreateMatcher() throws Exception { TestDataflowPipelineRunner runner = (TestDataflowPipelineRunner) p.getRunner(); p.getOptions().as(TestPipelineOptions.class) - .setOnCreateMatcher(new BaseMatcher() { + .setOnCreateMatcher(new CustomTypeSafeMatcher("TestMatcher") { @Override - public boolean matches(Object o) { - // Assert that job.WaitToFinish has not yet been called + protected boolean matchesSafely(PipelineResult pipelineResult) { try { verify(mockJob, never()).waitToFinish(any(Long.class), any(TimeUnit.class), any(JobMessagesHandler.class)); } catch (Exception e) { return false; } - return (PipelineResult) o == (PipelineResult) mockJob; - } - - @Override - public void describeTo(Description description) { + assertSame(mockJob, pipelineResult); + return true; } }); + when(mockJob.waitToFinish(any(Long.class), any(TimeUnit.class), any(JobMessagesHandler.class))) .thenReturn(State.DONE); @@ -466,7 +463,7 @@ public void describeTo(Description description) { } @Test - public void testBatchSuccessMatcherPipelineSuccess() throws Exception { + public void testBatchOnSuccessMatcherWhenPipelineSucceeds() throws Exception { Pipeline p = TestPipeline.create(options); PCollection pc = p.apply(Create.of(1, 2, 3)); PAssert.that(pc).containsInAnyOrder(1, 2, 3); @@ -482,21 +479,17 @@ public void testBatchSuccessMatcherPipelineSuccess() throws Exception { TestDataflowPipelineRunner runner = (TestDataflowPipelineRunner) p.getRunner(); p.getOptions().as(TestPipelineOptions.class) - .setOnSuccessMatcher(new BaseMatcher() { + .setOnSuccessMatcher(new CustomTypeSafeMatcher("TestMatcher") { @Override - public boolean matches(Object o) { - // Assert that job.WaitToFinish has not yet been called + protected boolean matchesSafely(PipelineResult pipelineResult) { try { verify(mockJob, Mockito.times(1)).waitToFinish(any(Long.class), any(TimeUnit.class), any(JobMessagesHandler.class)); } catch (Exception e) { return false; } - return (PipelineResult) o == (PipelineResult) mockJob; - } - - @Override - public void describeTo(Description description) { + assertSame(mockJob, pipelineResult); + return true; } }); @@ -506,7 +499,7 @@ public void describeTo(Description description) { } @Test - public void testStreamingSuccessMatcherPipelineSuccess() throws Exception { + public void testStreamingOnSuccessMatcherWhenPipelineSucceeds() throws Exception { options.setStreaming(true); Pipeline p = TestPipeline.create(options); PCollection pc = p.apply(Create.of(1, 2, 3)); @@ -523,23 +516,20 @@ public void testStreamingSuccessMatcherPipelineSuccess() throws Exception { TestDataflowPipelineRunner runner = (TestDataflowPipelineRunner) p.getRunner(); p.getOptions().as(TestPipelineOptions.class) - .setOnSuccessMatcher(new BaseMatcher() { + .setOnSuccessMatcher(new CustomTypeSafeMatcher("TestMatcher") { @Override - public boolean matches(Object o) { - // Assert that job.WaitToFinish has not yet been called + protected boolean matchesSafely(PipelineResult pipelineResult) { try { verify(mockJob, Mockito.times(1)).waitToFinish(any(Long.class), any(TimeUnit.class), any(JobMessagesHandler.class)); } catch (Exception e) { return false; } - return (PipelineResult) o == (PipelineResult) mockJob; - } - - @Override - public void describeTo(Description description) { + assertSame(mockJob, pipelineResult); + return true; } }); + when(mockJob.waitToFinish(any(Long.class), any(TimeUnit.class), any(JobMessagesHandler.class))) .thenReturn(State.DONE); @@ -549,7 +539,7 @@ public void describeTo(Description description) { } @Test - public void testBatchSuccessMatcherPipelineFailure() throws Exception { + public void testBatchOnSuccessMatcherWhenPipelineFails() throws Exception { Pipeline p = TestPipeline.create(options); PCollection pc = p.apply(Create.of(1, 2, 3)); PAssert.that(pc).containsInAnyOrder(1, 2, 3); @@ -565,14 +555,11 @@ public void testBatchSuccessMatcherPipelineFailure() throws Exception { TestDataflowPipelineRunner runner = (TestDataflowPipelineRunner) p.getRunner(); p.getOptions().as(TestPipelineOptions.class) - .setOnSuccessMatcher(new BaseMatcher() { + .setOnSuccessMatcher(new CustomTypeSafeMatcher("TestMatcher") { @Override - public boolean matches(Object o) { - throw new AssertionError("Success matcher shouldn't be called."); - } - - @Override - public void describeTo(Description description) { + protected boolean matchesSafely(PipelineResult pipelineResult) { + fail("OnSuccessMatcher should not be called on pipeline failure."); + return false; } }); @@ -589,7 +576,7 @@ public void describeTo(Description description) { } @Test - public void testStreamingSuccessMatcherPipelineFailure() throws Exception { + public void testStreamingOnSuccessMatcherWhenPipelineFails() throws Exception { options.setStreaming(true); Pipeline p = TestPipeline.create(options); PCollection pc = p.apply(Create.of(1, 2, 3)); @@ -606,16 +593,14 @@ public void testStreamingSuccessMatcherPipelineFailure() throws Exception { TestDataflowPipelineRunner runner = (TestDataflowPipelineRunner) p.getRunner(); p.getOptions().as(TestPipelineOptions.class) - .setOnSuccessMatcher(new BaseMatcher() { + .setOnSuccessMatcher(new CustomTypeSafeMatcher("TestMatcher") { @Override - public boolean matches(Object o) { - throw new AssertionError("Success matcher shouldn't be called."); - } - - @Override - public void describeTo(Description description) { + protected boolean matchesSafely(PipelineResult pipelineResult) { + fail("OnSuccessMatcher should not be called on pipeline failure."); + return false; } }); + when(mockJob.waitToFinish(any(Long.class), any(TimeUnit.class), any(JobMessagesHandler.class))) .thenReturn(State.FAILED); From 203639b17adce25959e9896d4b553ce7641e17eb Mon Sep 17 00:00:00 2001 From: Jason Kuster Date: Tue, 3 May 2016 14:06:07 -0700 Subject: [PATCH 4/8] Second wave of pull request fixes. Signed-off-by: Jason Kuster --- .../TestDataflowPipelineRunnerTest.java | 2 -- .../apache/beam/sdk/testing/TestPipeline.java | 19 ++++++++----------- .../beam/sdk/testing/TestPipelineOptions.java | 2 +- 3 files changed, 9 insertions(+), 14 deletions(-) diff --git a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunnerTest.java b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunnerTest.java index 8f43f3cd93a3..6adb21654c5d 100644 --- a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunnerTest.java +++ b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunnerTest.java @@ -65,9 +65,7 @@ import com.google.common.collect.ImmutableMap; import com.google.common.collect.Lists; -import org.hamcrest.BaseMatcher; import org.hamcrest.CustomTypeSafeMatcher; -import org.hamcrest.Description; import org.joda.time.Instant; import org.junit.Before; import org.junit.Rule; diff --git a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipeline.java b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipeline.java index 7c3a92ccfba8..beccb72d0014 100644 --- a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipeline.java +++ b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipeline.java @@ -50,7 +50,6 @@ import java.io.IOException; import java.lang.reflect.Method; import java.util.ArrayList; -import java.util.HashMap; import java.util.Iterator; import java.util.Map; import java.util.UUID; @@ -241,8 +240,13 @@ private static Optional findCallersStackTrace() { return firstInstanceAfterTestPipeline; } + /** + * IntegrationMatcherStore is an inner class of TestPipeline which is used for masking the + * serialization of Matchers. + */ private static class IntegrationMatcherStore extends Module { - private static Map> matcherMap = new ConcurrentHashMap<>(); + private static final Map> MATCHER_MAP = + new ConcurrentHashMap<>(); /** * Adds a matcher to the matcher store, to be retrieved by a TestPipelineRunner during @@ -253,11 +257,8 @@ private static class IntegrationMatcherStore extends Module { * opts.SetPassVerifier(matcher) */ String addMatcher(Matcher m) { - if (matcherMap == null) { - matcherMap = new HashMap<>(); - } String uid = UUID.randomUUID().toString(); - matcherMap.put(uid, m); + MATCHER_MAP.put(uid, m); return uid; } @@ -268,10 +269,7 @@ String addMatcher(Matcher m) { * @return Returns the matcher corresponding to the string passed, or null if it doesn't exist. */ Matcher getMatcher(String str) { - if (matcherMap == null) { - return null; - } - return matcherMap.get(str); + return MATCHER_MAP.get(str); } @Override @@ -304,7 +302,6 @@ public void serialize(Matcher matcher, JsonGenerator jsonGenerat } class MatcherDeserializer extends JsonDeserializer> { - @Override public Matcher deserialize(JsonParser jsonParser, DeserializationContext deserializationContext) diff --git a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipelineOptions.java b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipelineOptions.java index a15d612ef552..4149941c1e83 100644 --- a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipelineOptions.java +++ b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipelineOptions.java @@ -42,7 +42,7 @@ public interface TestPipelineOptions extends PipelineOptions { void setOnSuccessMatcher(Matcher value); /** - * Factory for PipelineResult matchers which always pass. + * Factory for {@link PipelineResult} matchers which always pass. */ class AlwaysPassMatcherFactory implements DefaultValueFactory> { From a3c71850c5b87aaaf8ab2209a620e660597c2e10 Mon Sep 17 00:00:00 2001 From: Jason Kuster Date: Tue, 3 May 2016 18:32:47 -0700 Subject: [PATCH 5/8] Redo...most things. Signed-off-by: Jason Kuster --- .../org/apache/beam/examples/WordCountIT.java | 7 +- .../beam/sdk/testing/MatcherDeserializer.java | 46 ++++++++ .../beam/sdk/testing/MatcherSerializer.java | 41 +++++++ .../beam/sdk/testing/SerializableMatcher.java | 8 +- .../apache/beam/sdk/testing/TestPipeline.java | 103 ++++-------------- .../beam/sdk/testing/TestPipelineOptions.java | 29 +++-- .../beam/sdk/testing/TestPipelineTest.java | 53 +++++++++ 7 files changed, 192 insertions(+), 95 deletions(-) create mode 100644 sdks/java/core/src/main/java/org/apache/beam/sdk/testing/MatcherDeserializer.java create mode 100644 sdks/java/core/src/main/java/org/apache/beam/sdk/testing/MatcherSerializer.java diff --git a/examples/java/src/test/java/org/apache/beam/examples/WordCountIT.java b/examples/java/src/test/java/org/apache/beam/examples/WordCountIT.java index 4d8c04b55f62..812c19ddc5ff 100644 --- a/examples/java/src/test/java/org/apache/beam/examples/WordCountIT.java +++ b/examples/java/src/test/java/org/apache/beam/examples/WordCountIT.java @@ -29,6 +29,8 @@ import org.junit.runner.RunWith; import org.junit.runners.JUnit4; +import java.util.Date; + /** * End-to-end tests of WordCount. */ @@ -38,8 +40,7 @@ public class WordCountIT { /** * Options for the WordCount Integration Test. */ - public static interface WordCountITOptions extends TestPipelineOptions, - WordCountOptions, DataflowPipelineOptions { + public interface WordCountITOptions extends TestPipelineOptions, WordCountOptions { } @Test @@ -47,7 +48,7 @@ public void testE2EWordCount() throws Exception { PipelineOptionsFactory.register(WordCountITOptions.class); WordCountITOptions options = TestPipeline.testingPipelineOptions().as(WordCountITOptions.class); options.setOutput(Joiner.on("/").join(new String[]{options.getTempRoot(), - options.getJobName(), "output", "results"})); + String.format("WordCountIT-%tF-%> { + @Override + public SerializableMatcher deserialize(JsonParser jsonParser, + DeserializationContext deserializationContext) + throws IOException, JsonProcessingException { + ObjectNode node = jsonParser.readValueAsTree(); + String matcher = node.get("matcher").asText(); + byte[] in = Base64.decodeBase64(matcher); + return (SerializableMatcher) SerializableUtils + .deserializeFromByteArray(in, "SerializableMatcher"); + } +} \ No newline at end of file diff --git a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/MatcherSerializer.java b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/MatcherSerializer.java new file mode 100644 index 000000000000..000433dcc0a8 --- /dev/null +++ b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/MatcherSerializer.java @@ -0,0 +1,41 @@ +/* + * 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. + */ +package org.apache.beam.sdk.testing; + +import com.fasterxml.jackson.core.JsonGenerator; +import com.fasterxml.jackson.core.JsonProcessingException; +import com.fasterxml.jackson.databind.JsonSerializer; +import com.fasterxml.jackson.databind.SerializerProvider; + +import org.apache.beam.sdk.util.SerializableUtils; + +import java.io.IOException; + +/** + * MatcherSerializer is used with Jackson to enable serialization of SerializableMatchers. + */ +class MatcherSerializer extends JsonSerializer> { + @Override + public void serialize(SerializableMatcher matcher, JsonGenerator jsonGenerator, + SerializerProvider serializerProvider) throws IOException, JsonProcessingException { + byte[] out = SerializableUtils.serializeToByteArray(matcher); + jsonGenerator.writeStartObject(); + jsonGenerator.writeBinaryField("matcher", out); + jsonGenerator.writeEndObject(); + } +} diff --git a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/SerializableMatcher.java b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/SerializableMatcher.java index 9132db78726e..2242eb3fd8d2 100644 --- a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/SerializableMatcher.java +++ b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/SerializableMatcher.java @@ -17,8 +17,11 @@ */ package org.apache.beam.sdk.testing; +import com.fasterxml.jackson.databind.annotation.JsonDeserialize; +import com.fasterxml.jackson.databind.annotation.JsonSerialize; import org.hamcrest.Matcher; +import java.io.IOException; import java.io.Serializable; /** @@ -32,5 +35,8 @@ * * @param The type of value matched. */ -interface SerializableMatcher extends Matcher, Serializable { +@JsonSerialize(using = MatcherSerializer.class) +@JsonDeserialize(using = MatcherDeserializer.class) +public interface SerializableMatcher extends Matcher, Serializable { } + diff --git a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipeline.java b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipeline.java index beccb72d0014..f88b620413f3 100644 --- a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipeline.java +++ b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipeline.java @@ -34,15 +34,21 @@ import com.fasterxml.jackson.core.JsonGenerator; import com.fasterxml.jackson.core.JsonParser; import com.fasterxml.jackson.core.JsonProcessingException; +import com.fasterxml.jackson.core.TreeNode; import com.fasterxml.jackson.core.Version; +import com.fasterxml.jackson.databind.AbstractTypeResolver; +import com.fasterxml.jackson.databind.DeserializationConfig; import com.fasterxml.jackson.databind.DeserializationContext; +import com.fasterxml.jackson.databind.JavaType; import com.fasterxml.jackson.databind.JsonDeserializer; +import com.fasterxml.jackson.databind.JsonNode; import com.fasterxml.jackson.databind.JsonSerializer; import com.fasterxml.jackson.databind.Module; import com.fasterxml.jackson.databind.ObjectMapper; import com.fasterxml.jackson.databind.SerializerProvider; import com.fasterxml.jackson.databind.module.SimpleDeserializers; import com.fasterxml.jackson.databind.module.SimpleSerializers; +import com.fasterxml.jackson.databind.node.ObjectNode; import org.hamcrest.Matcher; import org.junit.experimental.categories.Category; @@ -52,6 +58,7 @@ import java.util.ArrayList; import java.util.Iterator; import java.util.Map; +import java.util.Map.Entry; import java.util.UUID; import java.util.concurrent.ConcurrentHashMap; @@ -97,10 +104,6 @@ public class TestPipeline extends Pipeline { private static final String PROPERTY_BEAM_TEST_PIPELINE_OPTIONS = "beamTestPipelineOptions"; private static final ObjectMapper MAPPER = new ObjectMapper(); - static { - MAPPER.registerModule(new IntegrationMatcherStore()); - } - /** * Creates and returns a new test pipeline. * @@ -175,16 +178,24 @@ public static PipelineOptions testingPipelineOptions() { public static String[] convertToArgs(PipelineOptions options) { try { - Map stringOpts = (Map) MAPPER.readValue( - MAPPER.writeValueAsBytes(options), Map.class).get("options"); + byte[] opts = MAPPER.writeValueAsBytes(options); + JsonParser jsonParser = MAPPER.getFactory().createParser(opts); + TreeNode node = jsonParser.readValueAsTree(); + ObjectNode optsNode = (ObjectNode) node.get("options"); ArrayList optArrayList = new ArrayList<>(); - for (Map.Entry entry : stringOpts.entrySet()) { - optArrayList.add("--" + entry.getKey() + "=" + entry.getValue()); + Iterator> entries = optsNode.fields(); + while (entries.hasNext()) { + Entry entry = entries.next(); + if (entry.getValue().isTextual()) { + optArrayList.add("--" + entry.getKey() + "=" + entry.getValue().asText()); + } else { + optArrayList.add("--" + entry.getKey() + "=" + entry.getValue()); + } } return optArrayList.toArray(new String[optArrayList.size()]); - } catch (Exception e) { - return null; + } catch (IOException e) { + throw new IllegalStateException(e); } } @@ -239,76 +250,4 @@ private static Optional findCallersStackTrace() { } return firstInstanceAfterTestPipeline; } - - /** - * IntegrationMatcherStore is an inner class of TestPipeline which is used for masking the - * serialization of Matchers. - */ - private static class IntegrationMatcherStore extends Module { - private static final Map> MATCHER_MAP = - new ConcurrentHashMap<>(); - - /** - * Adds a matcher to the matcher store, to be retrieved by a TestPipelineRunner during - * pipeline execution. - * - * @param m The matcher to add to the store. - * @return Returns a UUID corresponding to the matcher which can be used in e.g. - * opts.SetPassVerifier(matcher) - */ - String addMatcher(Matcher m) { - String uid = UUID.randomUUID().toString(); - MATCHER_MAP.put(uid, m); - return uid; - } - - /** - * Retrieves a matcher from the matcher store. - * - * @param str The UUID of the matcher to return. - * @return Returns the matcher corresponding to the string passed, or null if it doesn't exist. - */ - Matcher getMatcher(String str) { - return MATCHER_MAP.get(str); - } - - @Override - public String getModuleName() { - return "IntegrationMatcherStore"; - } - - @Override - public Version version() { - return new Version(1, 0, 0, "", null, null); - } - - @Override - public void setupModule(SetupContext setupContext) { - SimpleSerializers ser = new SimpleSerializers(); - ser.addSerializer(Matcher.class, (JsonSerializer) new MatcherSerializer()); - SimpleDeserializers des = new SimpleDeserializers(); - des.addDeserializer(Matcher.class, new MatcherDeserializer()); - setupContext.addSerializers(ser); - setupContext.addDeserializers(des); - } - - class MatcherSerializer extends JsonSerializer> { - @Override - public void serialize(Matcher matcher, JsonGenerator jsonGenerator, - SerializerProvider serializerProvider) throws IOException, JsonProcessingException { - String uuid = addMatcher(matcher); - jsonGenerator.writeString(uuid); - } - } - - class MatcherDeserializer extends JsonDeserializer> { - @Override - public Matcher deserialize(JsonParser jsonParser, - DeserializationContext deserializationContext) - throws IOException, JsonProcessingException { - String uuid = jsonParser.getText(); - return getMatcher(uuid); - } - } - } } diff --git a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipelineOptions.java b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipelineOptions.java index 4149941c1e83..cc27a4973255 100644 --- a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipelineOptions.java +++ b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipelineOptions.java @@ -21,8 +21,8 @@ import org.apache.beam.sdk.options.Default; import org.apache.beam.sdk.options.DefaultValueFactory; import org.apache.beam.sdk.options.PipelineOptions; -import org.hamcrest.Matcher; -import org.hamcrest.Matchers; +import org.hamcrest.BaseMatcher; +import org.hamcrest.Description; /** * {@link TestPipelineOptions} is a set of options for test pipelines. @@ -34,22 +34,33 @@ public interface TestPipelineOptions extends PipelineOptions { void setTempRoot(String value); @Default.InstanceFactory(AlwaysPassMatcherFactory.class) - Matcher getOnCreateMatcher(); - void setOnCreateMatcher(Matcher value); + SerializableMatcher getOnCreateMatcher(); + void setOnCreateMatcher(SerializableMatcher value); @Default.InstanceFactory(AlwaysPassMatcherFactory.class) - Matcher getOnSuccessMatcher(); - void setOnSuccessMatcher(Matcher value); + SerializableMatcher getOnSuccessMatcher(); + void setOnSuccessMatcher(SerializableMatcher value); /** * Factory for {@link PipelineResult} matchers which always pass. */ class AlwaysPassMatcherFactory - implements DefaultValueFactory> { + implements DefaultValueFactory> { @Override - public Matcher create(PipelineOptions options) { - return Matchers.any(PipelineResult.class); + public SerializableMatcher create(PipelineOptions options) { + return new AlwaysPassMatcher(); } } + class AlwaysPassMatcher extends BaseMatcher + implements SerializableMatcher { + @Override + public boolean matches(Object o) { + return true; + } + + @Override + public void describeTo(Description description) { + } + } } diff --git a/sdks/java/core/src/test/java/org/apache/beam/sdk/testing/TestPipelineTest.java b/sdks/java/core/src/test/java/org/apache/beam/sdk/testing/TestPipelineTest.java index 9460e13d2c84..6320b926e0fa 100644 --- a/sdks/java/core/src/test/java/org/apache/beam/sdk/testing/TestPipelineTest.java +++ b/sdks/java/core/src/test/java/org/apache/beam/sdk/testing/TestPipelineTest.java @@ -21,8 +21,10 @@ import static org.hamcrest.Matchers.containsInAnyOrder; import static org.junit.Assert.assertEquals; import static org.junit.Assert.assertNotNull; +import static org.junit.Assert.assertSame; import static org.junit.Assert.assertThat; +import org.apache.beam.sdk.PipelineResult; import org.apache.beam.sdk.options.ApplicationNameOptions; import org.apache.beam.sdk.options.GcpOptions; import org.apache.beam.sdk.options.PipelineOptions; @@ -31,6 +33,10 @@ import com.fasterxml.jackson.databind.ObjectMapper; +import org.hamcrest.BaseMatcher; +import org.hamcrest.CustomTypeSafeMatcher; +import org.hamcrest.Description; +import org.hamcrest.Matcher; import org.junit.Rule; import org.junit.Test; import org.junit.rules.TestRule; @@ -38,7 +44,9 @@ import org.junit.runners.JUnit4; import java.util.Arrays; +import java.util.Date; import java.util.List; +import java.util.UUID; /** Tests for {@link TestPipeline}. */ @RunWith(JUnit4.class) @@ -116,4 +124,49 @@ public TestPipeline p() { return TestPipeline.create(); } } + + @Test + public void testMatcherSerializationDeserialization() { + TestPipelineOptions opts = PipelineOptionsFactory.as(TestPipelineOptions.class); + SerializableMatcher m1 = new TestMatcher(); + SerializableMatcher m2 = new TestMatcher(); + + opts.setOnCreateMatcher(m1); + opts.setOnSuccessMatcher(m2); + + String[] arr = TestPipeline.convertToArgs(opts); + TestPipelineOptions newOpts = PipelineOptionsFactory.fromArgs(arr) + .as(TestPipelineOptions.class); + + assertEquals(m1, newOpts.getOnCreateMatcher()); + assertEquals(m2, newOpts.getOnSuccessMatcher()); + } + + public static class TestMatcher extends BaseMatcher + implements SerializableMatcher { + final UUID uuid = UUID.randomUUID(); + @Override + public boolean matches(Object o) { + return true; + } + + @Override + public void describeTo(Description description) { + description.appendText(String.format("%tL", new Date())); + } + + @Override + public boolean equals(Object obj) { + if (!(obj instanceof TestMatcher)) { + return false; + } + TestMatcher other = (TestMatcher) obj; + return other.uuid.equals(uuid); + } + + @Override + public int hashCode() { + return uuid.hashCode(); + } + } } From c855b26f0391107e8b6b7dd9abeb52dd1819435a Mon Sep 17 00:00:00 2001 From: Jason Kuster Date: Tue, 3 May 2016 18:57:33 -0700 Subject: [PATCH 6/8] Checkstyle fixes. Signed-off-by: Jason Kuster --- .../beam/sdk/testing/MatcherDeserializer.java | 6 +++--- .../beam/sdk/testing/SerializableMatcher.java | 1 - .../apache/beam/sdk/testing/TestPipeline.java | 17 ----------------- .../beam/sdk/testing/TestPipelineOptions.java | 3 +++ .../beam/sdk/testing/TestPipelineTest.java | 3 --- 5 files changed, 6 insertions(+), 24 deletions(-) diff --git a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/MatcherDeserializer.java b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/MatcherDeserializer.java index d63823244566..c899453843a3 100644 --- a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/MatcherDeserializer.java +++ b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/MatcherDeserializer.java @@ -19,14 +19,14 @@ import com.google.api.client.util.Base64; +import org.apache.beam.sdk.util.SerializableUtils; + import com.fasterxml.jackson.core.JsonParser; import com.fasterxml.jackson.core.JsonProcessingException; import com.fasterxml.jackson.databind.DeserializationContext; import com.fasterxml.jackson.databind.JsonDeserializer; import com.fasterxml.jackson.databind.node.ObjectNode; -import org.apache.beam.sdk.util.SerializableUtils; - import java.io.IOException; /** @@ -43,4 +43,4 @@ public SerializableMatcher deserialize(JsonParser jsonParser, return (SerializableMatcher) SerializableUtils .deserializeFromByteArray(in, "SerializableMatcher"); } -} \ No newline at end of file +} diff --git a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/SerializableMatcher.java b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/SerializableMatcher.java index 2242eb3fd8d2..a465bbec32aa 100644 --- a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/SerializableMatcher.java +++ b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/SerializableMatcher.java @@ -21,7 +21,6 @@ import com.fasterxml.jackson.databind.annotation.JsonSerialize; import org.hamcrest.Matcher; -import java.io.IOException; import java.io.Serializable; /** diff --git a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipeline.java b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipeline.java index f88b620413f3..a4921d56be0a 100644 --- a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipeline.java +++ b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipeline.java @@ -31,36 +31,19 @@ import com.google.common.base.Strings; import com.google.common.collect.Iterators; -import com.fasterxml.jackson.core.JsonGenerator; import com.fasterxml.jackson.core.JsonParser; -import com.fasterxml.jackson.core.JsonProcessingException; import com.fasterxml.jackson.core.TreeNode; -import com.fasterxml.jackson.core.Version; -import com.fasterxml.jackson.databind.AbstractTypeResolver; -import com.fasterxml.jackson.databind.DeserializationConfig; -import com.fasterxml.jackson.databind.DeserializationContext; -import com.fasterxml.jackson.databind.JavaType; -import com.fasterxml.jackson.databind.JsonDeserializer; import com.fasterxml.jackson.databind.JsonNode; -import com.fasterxml.jackson.databind.JsonSerializer; -import com.fasterxml.jackson.databind.Module; import com.fasterxml.jackson.databind.ObjectMapper; -import com.fasterxml.jackson.databind.SerializerProvider; -import com.fasterxml.jackson.databind.module.SimpleDeserializers; -import com.fasterxml.jackson.databind.module.SimpleSerializers; import com.fasterxml.jackson.databind.node.ObjectNode; -import org.hamcrest.Matcher; import org.junit.experimental.categories.Category; import java.io.IOException; import java.lang.reflect.Method; import java.util.ArrayList; import java.util.Iterator; -import java.util.Map; import java.util.Map.Entry; -import java.util.UUID; -import java.util.concurrent.ConcurrentHashMap; import javax.annotation.Nullable; diff --git a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipelineOptions.java b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipelineOptions.java index cc27a4973255..ff553bafa85e 100644 --- a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipelineOptions.java +++ b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipelineOptions.java @@ -52,6 +52,9 @@ public SerializableMatcher create(PipelineOptions options) { } } + /** + * Matcher which will always pass. + */ class AlwaysPassMatcher extends BaseMatcher implements SerializableMatcher { @Override diff --git a/sdks/java/core/src/test/java/org/apache/beam/sdk/testing/TestPipelineTest.java b/sdks/java/core/src/test/java/org/apache/beam/sdk/testing/TestPipelineTest.java index 6320b926e0fa..28ed9433d336 100644 --- a/sdks/java/core/src/test/java/org/apache/beam/sdk/testing/TestPipelineTest.java +++ b/sdks/java/core/src/test/java/org/apache/beam/sdk/testing/TestPipelineTest.java @@ -21,7 +21,6 @@ import static org.hamcrest.Matchers.containsInAnyOrder; import static org.junit.Assert.assertEquals; import static org.junit.Assert.assertNotNull; -import static org.junit.Assert.assertSame; import static org.junit.Assert.assertThat; import org.apache.beam.sdk.PipelineResult; @@ -34,9 +33,7 @@ import com.fasterxml.jackson.databind.ObjectMapper; import org.hamcrest.BaseMatcher; -import org.hamcrest.CustomTypeSafeMatcher; import org.hamcrest.Description; -import org.hamcrest.Matcher; import org.junit.Rule; import org.junit.Test; import org.junit.rules.TestRule; From d5915e9b7da07c437f7f8bdf8bb322421c688543 Mon Sep 17 00:00:00 2001 From: Jason Kuster Date: Tue, 3 May 2016 19:08:15 -0700 Subject: [PATCH 7/8] First attempt at fixing TestDataflowPipelineRunnerTest Signed-off-by: Jason Kuster --- .../org/apache/beam/examples/WordCountIT.java | 1 - .../TestDataflowPipelineRunnerTest.java | 120 ++++++++---------- .../beam/sdk/testing/MatcherDeserializer.java | 4 +- .../beam/sdk/testing/MatcherSerializer.java | 4 +- .../beam/sdk/testing/TestPipelineTest.java | 3 + 5 files changed, 59 insertions(+), 73 deletions(-) diff --git a/examples/java/src/test/java/org/apache/beam/examples/WordCountIT.java b/examples/java/src/test/java/org/apache/beam/examples/WordCountIT.java index 812c19ddc5ff..a09ec02881b0 100644 --- a/examples/java/src/test/java/org/apache/beam/examples/WordCountIT.java +++ b/examples/java/src/test/java/org/apache/beam/examples/WordCountIT.java @@ -19,7 +19,6 @@ package org.apache.beam.examples; import org.apache.beam.examples.WordCount.WordCountOptions; -import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; import org.apache.beam.sdk.options.PipelineOptionsFactory; import org.apache.beam.sdk.testing.TestPipeline; import org.apache.beam.sdk.testing.TestPipelineOptions; diff --git a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunnerTest.java b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunnerTest.java index 6adb21654c5d..b8bd44d03843 100644 --- a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunnerTest.java +++ b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunnerTest.java @@ -27,7 +27,6 @@ import static org.mockito.Mockito.atLeastOnce; import static org.mockito.Mockito.doCallRealMethod; import static org.mockito.Mockito.doReturn; -import static org.mockito.Mockito.never; import static org.mockito.Mockito.spy; import static org.mockito.Mockito.verify; import static org.mockito.Mockito.when; @@ -42,6 +41,7 @@ import org.apache.beam.sdk.PipelineResult.State; import org.apache.beam.sdk.options.PipelineOptionsFactory; import org.apache.beam.sdk.testing.PAssert; +import org.apache.beam.sdk.testing.SerializableMatcher; import org.apache.beam.sdk.testing.TestPipeline; import org.apache.beam.sdk.testing.TestPipelineOptions; import org.apache.beam.sdk.transforms.Create; @@ -65,7 +65,8 @@ import com.google.common.collect.ImmutableMap; import com.google.common.collect.Lists; -import org.hamcrest.CustomTypeSafeMatcher; +import org.hamcrest.BaseMatcher; +import org.hamcrest.Description; import org.joda.time.Instant; import org.junit.Before; import org.junit.Rule; @@ -401,19 +402,7 @@ public void testBatchOnCreateMatcher() throws Exception { TestDataflowPipelineRunner runner = (TestDataflowPipelineRunner) p.getRunner(); p.getOptions().as(TestPipelineOptions.class) - .setOnCreateMatcher(new CustomTypeSafeMatcher("TestMatcher") { - @Override - protected boolean matchesSafely(PipelineResult pipelineResult) { - try { - verify(mockJob, never()).waitToFinish(any(Long.class), any(TimeUnit.class), - any(JobMessagesHandler.class)); - } catch (Exception e) { - return false; - } - assertSame(mockJob, pipelineResult); - return true; - } - }); + .setOnCreateMatcher(new TestSuccessMatcher(mockJob, 0)); when(request.execute()).thenReturn( generateMockMetricResponse(true /* success */, true /* tentative */)); @@ -438,19 +427,7 @@ public void testStreamingOnCreateMatcher() throws Exception { TestDataflowPipelineRunner runner = (TestDataflowPipelineRunner) p.getRunner(); p.getOptions().as(TestPipelineOptions.class) - .setOnCreateMatcher(new CustomTypeSafeMatcher("TestMatcher") { - @Override - protected boolean matchesSafely(PipelineResult pipelineResult) { - try { - verify(mockJob, never()).waitToFinish(any(Long.class), any(TimeUnit.class), - any(JobMessagesHandler.class)); - } catch (Exception e) { - return false; - } - assertSame(mockJob, pipelineResult); - return true; - } - }); + .setOnCreateMatcher(new TestSuccessMatcher(mockJob, 0)); when(mockJob.waitToFinish(any(Long.class), any(TimeUnit.class), any(JobMessagesHandler.class))) .thenReturn(State.DONE); @@ -477,19 +454,7 @@ public void testBatchOnSuccessMatcherWhenPipelineSucceeds() throws Exception { TestDataflowPipelineRunner runner = (TestDataflowPipelineRunner) p.getRunner(); p.getOptions().as(TestPipelineOptions.class) - .setOnSuccessMatcher(new CustomTypeSafeMatcher("TestMatcher") { - @Override - protected boolean matchesSafely(PipelineResult pipelineResult) { - try { - verify(mockJob, Mockito.times(1)).waitToFinish(any(Long.class), any(TimeUnit.class), - any(JobMessagesHandler.class)); - } catch (Exception e) { - return false; - } - assertSame(mockJob, pipelineResult); - return true; - } - }); + .setOnSuccessMatcher(new TestSuccessMatcher(mockJob, 1)); when(request.execute()).thenReturn( generateMockMetricResponse(true /* success */, true /* tentative */)); @@ -514,19 +479,7 @@ public void testStreamingOnSuccessMatcherWhenPipelineSucceeds() throws Exception TestDataflowPipelineRunner runner = (TestDataflowPipelineRunner) p.getRunner(); p.getOptions().as(TestPipelineOptions.class) - .setOnSuccessMatcher(new CustomTypeSafeMatcher("TestMatcher") { - @Override - protected boolean matchesSafely(PipelineResult pipelineResult) { - try { - verify(mockJob, Mockito.times(1)).waitToFinish(any(Long.class), any(TimeUnit.class), - any(JobMessagesHandler.class)); - } catch (Exception e) { - return false; - } - assertSame(mockJob, pipelineResult); - return true; - } - }); + .setOnSuccessMatcher(new TestSuccessMatcher(mockJob, 1)); when(mockJob.waitToFinish(any(Long.class), any(TimeUnit.class), any(JobMessagesHandler.class))) .thenReturn(State.DONE); @@ -553,13 +506,7 @@ public void testBatchOnSuccessMatcherWhenPipelineFails() throws Exception { TestDataflowPipelineRunner runner = (TestDataflowPipelineRunner) p.getRunner(); p.getOptions().as(TestPipelineOptions.class) - .setOnSuccessMatcher(new CustomTypeSafeMatcher("TestMatcher") { - @Override - protected boolean matchesSafely(PipelineResult pipelineResult) { - fail("OnSuccessMatcher should not be called on pipeline failure."); - return false; - } - }); + .setOnSuccessMatcher(new TestFailureMatcher()); when(request.execute()).thenReturn( generateMockMetricResponse(false /* success */, true /* tentative */)); @@ -591,13 +538,7 @@ public void testStreamingOnSuccessMatcherWhenPipelineFails() throws Exception { TestDataflowPipelineRunner runner = (TestDataflowPipelineRunner) p.getRunner(); p.getOptions().as(TestPipelineOptions.class) - .setOnSuccessMatcher(new CustomTypeSafeMatcher("TestMatcher") { - @Override - protected boolean matchesSafely(PipelineResult pipelineResult) { - fail("OnSuccessMatcher should not be called on pipeline failure."); - return false; - } - }); + .setOnSuccessMatcher(new TestFailureMatcher()); when(mockJob.waitToFinish(any(Long.class), any(TimeUnit.class), any(JobMessagesHandler.class))) .thenReturn(State.FAILED); @@ -613,4 +554,47 @@ protected boolean matchesSafely(PipelineResult pipelineResult) { } fail("Expected an exception on pipeline failure."); } + + static class TestSuccessMatcher extends BaseMatcher implements + SerializableMatcher { + DataflowPipelineJob mockJob; + int called; + public TestSuccessMatcher(DataflowPipelineJob job, int times) { + this.mockJob = job; + this.called = times; + } + + @Override + public boolean matches(Object o) { + if (!(o instanceof PipelineResult)) { + fail(); + } + PipelineResult other = (PipelineResult) o; + try { + verify(mockJob, Mockito.times(called)).waitToFinish(any(Long.class), any(TimeUnit.class), + any(JobMessagesHandler.class)); + } catch (Exception e) { + return false; + } + assertSame(mockJob, other); + return true; + } + + @Override + public void describeTo(Description description) { + } + } + + static class TestFailureMatcher extends BaseMatcher implements + SerializableMatcher { + @Override + public boolean matches(Object o) { + fail("OnSuccessMatcher should not be called on pipeline failure."); + return false; + } + + @Override + public void describeTo(Description description) { + } + } } diff --git a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/MatcherDeserializer.java b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/MatcherDeserializer.java index c899453843a3..84984709bc64 100644 --- a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/MatcherDeserializer.java +++ b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/MatcherDeserializer.java @@ -17,10 +17,10 @@ */ package org.apache.beam.sdk.testing; -import com.google.api.client.util.Base64; - import org.apache.beam.sdk.util.SerializableUtils; +import com.google.api.client.util.Base64; + import com.fasterxml.jackson.core.JsonParser; import com.fasterxml.jackson.core.JsonProcessingException; import com.fasterxml.jackson.databind.DeserializationContext; diff --git a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/MatcherSerializer.java b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/MatcherSerializer.java index 000433dcc0a8..6154dccf24a2 100644 --- a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/MatcherSerializer.java +++ b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/MatcherSerializer.java @@ -17,13 +17,13 @@ */ package org.apache.beam.sdk.testing; +import org.apache.beam.sdk.util.SerializableUtils; + import com.fasterxml.jackson.core.JsonGenerator; import com.fasterxml.jackson.core.JsonProcessingException; import com.fasterxml.jackson.databind.JsonSerializer; import com.fasterxml.jackson.databind.SerializerProvider; -import org.apache.beam.sdk.util.SerializableUtils; - import java.io.IOException; /** diff --git a/sdks/java/core/src/test/java/org/apache/beam/sdk/testing/TestPipelineTest.java b/sdks/java/core/src/test/java/org/apache/beam/sdk/testing/TestPipelineTest.java index 28ed9433d336..5ece7fe62254 100644 --- a/sdks/java/core/src/test/java/org/apache/beam/sdk/testing/TestPipelineTest.java +++ b/sdks/java/core/src/test/java/org/apache/beam/sdk/testing/TestPipelineTest.java @@ -139,6 +139,9 @@ public void testMatcherSerializationDeserialization() { assertEquals(m2, newOpts.getOnSuccessMatcher()); } + /** + * TestMatcher is a matcher designed for testing matcher serialization/deserialization. + */ public static class TestMatcher extends BaseMatcher implements SerializableMatcher { final UUID uuid = UUID.randomUUID(); From ab604cf8deb51be95523502f729b2855ec135ca7 Mon Sep 17 00:00:00 2001 From: Jason Kuster Date: Tue, 3 May 2016 19:47:18 -0700 Subject: [PATCH 8/8] Final PR comment fixes. Signed-off-by: Jason Kuster --- .../testing/TestDataflowPipelineRunnerTest.java | 15 ++++++++------- .../beam/sdk/testing/MatcherSerializer.java | 5 ++++- .../apache/beam/sdk/testing/TestPipelineTest.java | 2 +- 3 files changed, 13 insertions(+), 9 deletions(-) diff --git a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunnerTest.java b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunnerTest.java index b8bd44d03843..fbaf116d98cb 100644 --- a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunnerTest.java +++ b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunnerTest.java @@ -80,6 +80,7 @@ import org.mockito.invocation.InvocationOnMock; import org.mockito.stubbing.Answer; +import java.io.IOException; import java.math.BigDecimal; import java.util.Arrays; import java.util.concurrent.TimeUnit; @@ -557,8 +558,9 @@ public void testStreamingOnSuccessMatcherWhenPipelineFails() throws Exception { static class TestSuccessMatcher extends BaseMatcher implements SerializableMatcher { - DataflowPipelineJob mockJob; - int called; + private final DataflowPipelineJob mockJob; + private final int called; + public TestSuccessMatcher(DataflowPipelineJob job, int times) { this.mockJob = job; this.called = times; @@ -567,16 +569,15 @@ public TestSuccessMatcher(DataflowPipelineJob job, int times) { @Override public boolean matches(Object o) { if (!(o instanceof PipelineResult)) { - fail(); + fail(String.format("Expected PipelineResult but received %s", o)); } - PipelineResult other = (PipelineResult) o; try { verify(mockJob, Mockito.times(called)).waitToFinish(any(Long.class), any(TimeUnit.class), any(JobMessagesHandler.class)); - } catch (Exception e) { - return false; + } catch (IOException | InterruptedException e) { + throw new AssertionError(e); } - assertSame(mockJob, other); + assertSame(mockJob, o); return true; } diff --git a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/MatcherSerializer.java b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/MatcherSerializer.java index 6154dccf24a2..0feeae06ad05 100644 --- a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/MatcherSerializer.java +++ b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/MatcherSerializer.java @@ -17,6 +17,8 @@ */ package org.apache.beam.sdk.testing; +import com.google.api.client.util.Base64; + import org.apache.beam.sdk.util.SerializableUtils; import com.fasterxml.jackson.core.JsonGenerator; @@ -34,8 +36,9 @@ class MatcherSerializer extends JsonSerializer> { public void serialize(SerializableMatcher matcher, JsonGenerator jsonGenerator, SerializerProvider serializerProvider) throws IOException, JsonProcessingException { byte[] out = SerializableUtils.serializeToByteArray(matcher); + String encodedString = Base64.encodeBase64String(out); jsonGenerator.writeStartObject(); - jsonGenerator.writeBinaryField("matcher", out); + jsonGenerator.writeStringField("matcher", encodedString); jsonGenerator.writeEndObject(); } } diff --git a/sdks/java/core/src/test/java/org/apache/beam/sdk/testing/TestPipelineTest.java b/sdks/java/core/src/test/java/org/apache/beam/sdk/testing/TestPipelineTest.java index 5ece7fe62254..8af4ff25bb06 100644 --- a/sdks/java/core/src/test/java/org/apache/beam/sdk/testing/TestPipelineTest.java +++ b/sdks/java/core/src/test/java/org/apache/beam/sdk/testing/TestPipelineTest.java @@ -144,7 +144,7 @@ public void testMatcherSerializationDeserialization() { */ public static class TestMatcher extends BaseMatcher implements SerializableMatcher { - final UUID uuid = UUID.randomUUID(); + private final UUID uuid = UUID.randomUUID(); @Override public boolean matches(Object o) { return true;