Return-Path: X-Original-To: archive-asf-public-internal@cust-asf2.ponee.io Delivered-To: archive-asf-public-internal@cust-asf2.ponee.io Received: from cust-asf.ponee.io (cust-asf.ponee.io [163.172.22.183]) by cust-asf2.ponee.io (Postfix) with ESMTP id 2E229200C15 for ; Tue, 24 Jan 2017 23:02:28 +0100 (CET) Received: by cust-asf.ponee.io (Postfix) id 2A065160B3E; Tue, 24 Jan 2017 22:02:28 +0000 (UTC) Delivered-To: archive-asf-public@cust-asf.ponee.io Received: from mail.apache.org (hermes.apache.org [140.211.11.3]) by cust-asf.ponee.io (Postfix) with SMTP id 56C46160B38 for ; Tue, 24 Jan 2017 23:02:27 +0100 (CET) Received: (qmail 16226 invoked by uid 500); 24 Jan 2017 22:02:26 -0000 Mailing-List: contact commits-help@beam.apache.org; run by ezmlm Precedence: bulk List-Help: List-Unsubscribe: List-Post: List-Id: Reply-To: dev@beam.apache.org Delivered-To: mailing list commits@beam.apache.org Received: (qmail 16217 invoked by uid 99); 24 Jan 2017 22:02:26 -0000 Received: from git1-us-west.apache.org (HELO git1-us-west.apache.org) (140.211.11.23) by apache.org (qpsmtpd/0.29) with ESMTP; Tue, 24 Jan 2017 22:02:26 +0000 Received: by git1-us-west.apache.org (ASF Mail Server at git1-us-west.apache.org, from userid 33) id 7692EDFB0E; Tue, 24 Jan 2017 22:02:26 +0000 (UTC) Content-Type: text/plain; charset="us-ascii" MIME-Version: 1.0 Content-Transfer-Encoding: 7bit From: kenn@apache.org To: commits@beam.apache.org Date: Tue, 24 Jan 2017 22:02:26 -0000 Message-Id: X-Mailer: ASF-Git Admin Mailer Subject: [1/2] beam git commit: Removes ReduceFnExecutor interface archived-at: Tue, 24 Jan 2017 22:02:28 -0000 Repository: beam Updated Branches: refs/heads/master b3334879f -> 11c3cd70b Removes ReduceFnExecutor interface Project: http://git-wip-us.apache.org/repos/asf/beam/repo Commit: http://git-wip-us.apache.org/repos/asf/beam/commit/8989473b Tree: http://git-wip-us.apache.org/repos/asf/beam/tree/8989473b Diff: http://git-wip-us.apache.org/repos/asf/beam/diff/8989473b Branch: refs/heads/master Commit: 8989473b8e379a40b888565aadead001379c9398 Parents: b333487 Author: Eugene Kirpichov Authored: Tue Jan 24 13:32:24 2017 -0800 Committer: Eugene Kirpichov Committed: Tue Jan 24 13:32:24 2017 -0800 ---------------------------------------------------------------------- .../apache/beam/runners/core/DoFnRunner.java | 20 -------------------- .../core/GroupAlsoByWindowViaWindowSetDoFn.java | 5 +---- .../beam/runners/direct/ParDoEvaluator.java | 2 -- .../runners/spark/translation/DoFnFunction.java | 2 -- .../spark/translation/MultiDoFnFunction.java | 2 -- 5 files changed, 1 insertion(+), 30 deletions(-) ---------------------------------------------------------------------- http://git-wip-us.apache.org/repos/asf/beam/blob/8989473b/runners/core-java/src/main/java/org/apache/beam/runners/core/DoFnRunner.java ---------------------------------------------------------------------- diff --git a/runners/core-java/src/main/java/org/apache/beam/runners/core/DoFnRunner.java b/runners/core-java/src/main/java/org/apache/beam/runners/core/DoFnRunner.java index 66f95db..b29adcc 100644 --- a/runners/core-java/src/main/java/org/apache/beam/runners/core/DoFnRunner.java +++ b/runners/core-java/src/main/java/org/apache/beam/runners/core/DoFnRunner.java @@ -17,12 +17,10 @@ */ package org.apache.beam.runners.core; -import org.apache.beam.sdk.transforms.Aggregator; import org.apache.beam.sdk.transforms.DoFn; import org.apache.beam.sdk.transforms.windowing.BoundedWindow; import org.apache.beam.sdk.util.TimeDomain; import org.apache.beam.sdk.util.WindowedValue; -import org.apache.beam.sdk.values.KV; import org.joda.time.Instant; /** @@ -51,22 +49,4 @@ public interface DoFnRunner { * additional tasks, such as flushing in-memory states. */ void finishBundle(); - - /** - * An internal interface for signaling that a {@link OldDoFn} requires late data dropping. - */ - public interface ReduceFnExecutor { - /** - * Gets this object as a {@link OldDoFn}. - * - *

Most implementors of this interface are expected to be {@link OldDoFn} instances, and will - * return themselves. - */ - OldDoFn, KV> asDoFn(); - - /** - * Returns an aggregator that tracks elements that are dropped due to being late. - */ - Aggregator getDroppedDueToLatenessAggregator(); - } } http://git-wip-us.apache.org/repos/asf/beam/blob/8989473b/runners/core-java/src/main/java/org/apache/beam/runners/core/GroupAlsoByWindowViaWindowSetDoFn.java ---------------------------------------------------------------------- diff --git a/runners/core-java/src/main/java/org/apache/beam/runners/core/GroupAlsoByWindowViaWindowSetDoFn.java b/runners/core-java/src/main/java/org/apache/beam/runners/core/GroupAlsoByWindowViaWindowSetDoFn.java index ecce4fc..d0387cf 100644 --- a/runners/core-java/src/main/java/org/apache/beam/runners/core/GroupAlsoByWindowViaWindowSetDoFn.java +++ b/runners/core-java/src/main/java/org/apache/beam/runners/core/GroupAlsoByWindowViaWindowSetDoFn.java @@ -17,7 +17,6 @@ */ package org.apache.beam.runners.core; -import org.apache.beam.runners.core.DoFnRunner.ReduceFnExecutor; import org.apache.beam.runners.core.triggers.ExecutableTriggerStateMachine; import org.apache.beam.runners.core.triggers.TriggerStateMachines; import org.apache.beam.sdk.transforms.Aggregator; @@ -37,7 +36,7 @@ import org.apache.beam.sdk.values.KV; @SystemDoFnInternal public class GroupAlsoByWindowViaWindowSetDoFn< K, InputT, OutputT, W extends BoundedWindow, RinT extends KeyedWorkItem> - extends OldDoFn> implements ReduceFnExecutor { + extends OldDoFn> { public static OldDoFn, KV> create( @@ -95,7 +94,6 @@ public class GroupAlsoByWindowViaWindowSetDoFn< reduceFnRunner.persist(); } - @Override public OldDoFn, KV> asDoFn() { // Safe contravariant cast @SuppressWarnings("unchecked") @@ -104,7 +102,6 @@ public class GroupAlsoByWindowViaWindowSetDoFn< return asFn; } - @Override public Aggregator getDroppedDueToLatenessAggregator() { return droppedDueToLateness; } http://git-wip-us.apache.org/repos/asf/beam/blob/8989473b/runners/direct-java/src/main/java/org/apache/beam/runners/direct/ParDoEvaluator.java ---------------------------------------------------------------------- diff --git a/runners/direct-java/src/main/java/org/apache/beam/runners/direct/ParDoEvaluator.java b/runners/direct-java/src/main/java/org/apache/beam/runners/direct/ParDoEvaluator.java index 97d5360..48f0f8d 100644 --- a/runners/direct-java/src/main/java/org/apache/beam/runners/direct/ParDoEvaluator.java +++ b/runners/direct-java/src/main/java/org/apache/beam/runners/direct/ParDoEvaluator.java @@ -73,8 +73,6 @@ class ParDoEvaluator implements TransformEvaluator { ReadyCheckingSideInputReader sideInputReader = evaluationContext.createSideInputReader(sideInputs); - // Unlike for OldDoFn, there is no ReduceFnExecutor that is a new DoFn, - // and window-exploded processing is achieved within the simple runner DoFnRunner underlying = DoFnRunners.simpleRunner( evaluationContext.getPipelineOptions(), http://git-wip-us.apache.org/repos/asf/beam/blob/8989473b/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/DoFnFunction.java ---------------------------------------------------------------------- diff --git a/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/DoFnFunction.java b/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/DoFnFunction.java index bd6cfbe..4fd5e51 100644 --- a/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/DoFnFunction.java +++ b/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/DoFnFunction.java @@ -81,8 +81,6 @@ public class DoFnFunction DoFnOutputManager outputManager = new DoFnOutputManager(); - // Unlike for OldDoFn, there is no ReduceFnExecutor that is a new DoFn, - // and window-exploded processing is achieved within the simple runner DoFnRunner doFnRunner = DoFnRunners.simpleRunner( runtimeContext.getPipelineOptions(), http://git-wip-us.apache.org/repos/asf/beam/blob/8989473b/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/MultiDoFnFunction.java ---------------------------------------------------------------------- diff --git a/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/MultiDoFnFunction.java b/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/MultiDoFnFunction.java index cceffc8..911e6c5 100644 --- a/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/MultiDoFnFunction.java +++ b/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/MultiDoFnFunction.java @@ -88,8 +88,6 @@ public class MultiDoFnFunction DoFnOutputManager outputManager = new DoFnOutputManager(); - // Unlike for OldDoFn, there is no ReduceFnExecutor that is a new DoFn, - // and window-exploded processing is achieved within the simple runner DoFnRunner doFnRunner = DoFnRunners.simpleRunner( runtimeContext.getPipelineOptions(),