FlinkRunner: remove the unused ReflectiveOneToOneOverrideFactory.
Project: http://git-wip-us.apache.org/repos/asf/beam/repo Commit: http://git-wip-us.apache.org/repos/asf/beam/commit/f69e3b53 Tree: http://git-wip-us.apache.org/repos/asf/beam/tree/f69e3b53 Diff: http://git-wip-us.apache.org/repos/asf/beam/diff/f69e3b53 Branch: refs/heads/master Commit: f69e3b53fafa4b79b21095d4b65edbe7cfeb7d2a Parents: e4ef23e Author: Pei He <[email protected]> Authored: Mon Jun 19 15:55:48 2017 -0700 Committer: Pei He <[email protected]> Committed: Tue Jun 20 14:12:13 2017 -0700 ---------------------------------------------------------------------- .../flink/FlinkStreamingPipelineTranslator.java | 31 -------------------- 1 file changed, 31 deletions(-) ---------------------------------------------------------------------- http://git-wip-us.apache.org/repos/asf/beam/blob/f69e3b53/runners/flink/src/main/java/org/apache/beam/runners/flink/FlinkStreamingPipelineTranslator.java ---------------------------------------------------------------------- diff --git a/runners/flink/src/main/java/org/apache/beam/runners/flink/FlinkStreamingPipelineTranslator.java b/runners/flink/src/main/java/org/apache/beam/runners/flink/FlinkStreamingPipelineTranslator.java index a88ff07..d768b01 100644 --- a/runners/flink/src/main/java/org/apache/beam/runners/flink/FlinkStreamingPipelineTranslator.java +++ b/runners/flink/src/main/java/org/apache/beam/runners/flink/FlinkStreamingPipelineTranslator.java @@ -24,7 +24,6 @@ import org.apache.beam.runners.core.SplittableParDoViaKeyedWorkItems; import org.apache.beam.runners.core.construction.PTransformMatchers; import org.apache.beam.runners.core.construction.PTransformReplacements; import org.apache.beam.runners.core.construction.ReplacementOutputs; -import org.apache.beam.runners.core.construction.SingleInputOutputOverrideFactory; import org.apache.beam.runners.core.construction.SplittableParDo; import org.apache.beam.runners.core.construction.UnconsumedReads; import org.apache.beam.sdk.Pipeline; @@ -36,7 +35,6 @@ import org.apache.beam.sdk.runners.TransformHierarchy; import org.apache.beam.sdk.transforms.PTransform; import org.apache.beam.sdk.transforms.ParDo.MultiOutput; import org.apache.beam.sdk.transforms.View.CreatePCollectionView; -import org.apache.beam.sdk.util.InstanceBuilder; import org.apache.beam.sdk.values.PCollection; import org.apache.beam.sdk.values.PCollectionTuple; import org.apache.beam.sdk.values.PValue; @@ -198,35 +196,6 @@ class FlinkStreamingPipelineTranslator extends FlinkPipelineTranslator { } } - private static class ReflectiveOneToOneOverrideFactory< - InputT, OutputT, TransformT extends PTransform<PCollection<InputT>, PCollection<OutputT>>> - extends SingleInputOutputOverrideFactory< - PCollection<InputT>, PCollection<OutputT>, TransformT> { - private final Class<PTransform<PCollection<InputT>, PCollection<OutputT>>> replacement; - private final FlinkRunner runner; - - private ReflectiveOneToOneOverrideFactory( - Class<PTransform<PCollection<InputT>, PCollection<OutputT>>> replacement, - FlinkRunner runner) { - this.replacement = replacement; - this.runner = runner; - } - - @Override - public PTransformReplacement<PCollection<InputT>, PCollection<OutputT>> getReplacementTransform( - AppliedPTransform<PCollection<InputT>, PCollection<OutputT>, TransformT> transform) { - return PTransformReplacement.of( - PTransformReplacements.getSingletonMainInput(transform), - InstanceBuilder.ofType(replacement) - .withArg(FlinkRunner.class, runner) - .withArg( - (Class<PTransform<PCollection<InputT>, PCollection<OutputT>>>) - transform.getTransform().getClass(), - transform.getTransform()) - .build()); - } - } - /** * A {@link PTransformOverrideFactory} that overrides a <a * href="https://s.apache.org/splittable-do-fn">Splittable DoFn</a> with {@link SplittableParDo}.
