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 4E3B8200C2E for ; Sun, 5 Mar 2017 16:52:33 +0100 (CET) Received: by cust-asf.ponee.io (Postfix) id 4CE90160B7D; Sun, 5 Mar 2017 15:52:33 +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 9F407160B65 for ; Sun, 5 Mar 2017 16:52:32 +0100 (CET) Received: (qmail 61419 invoked by uid 500); 5 Mar 2017 15:52:31 -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 61409 invoked by uid 99); 5 Mar 2017 15:52:31 -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; Sun, 05 Mar 2017 15:52:31 +0000 Received: by git1-us-west.apache.org (ASF Mail Server at git1-us-west.apache.org, from userid 33) id AD3DADFA1F; Sun, 5 Mar 2017 15:52:31 +0000 (UTC) Content-Type: text/plain; charset="us-ascii" MIME-Version: 1.0 Content-Transfer-Encoding: 7bit From: staslevin@apache.org To: commits@beam.apache.org Date: Sun, 05 Mar 2017 15:52:31 -0000 Message-Id: X-Mailer: ASF-Git Admin Mailer Subject: [1/2] beam git commit: [BEAM-1625] BoundedDataset action() does not materialize RDD archived-at: Sun, 05 Mar 2017 15:52:33 -0000 Repository: beam Updated Branches: refs/heads/master 11a381b23 -> 59451bca6 [BEAM-1625] BoundedDataset action() does not materialize RDD Project: http://git-wip-us.apache.org/repos/asf/beam/repo Commit: http://git-wip-us.apache.org/repos/asf/beam/commit/4febd954 Tree: http://git-wip-us.apache.org/repos/asf/beam/tree/4febd954 Diff: http://git-wip-us.apache.org/repos/asf/beam/diff/4febd954 Branch: refs/heads/master Commit: 4febd954af00458032efbee45b7f9724fe0ea9ed Parents: 11a381b Author: Aviem Zur Authored: Sun Mar 5 16:17:35 2017 +0200 Committer: Aviem Zur Committed: Sun Mar 5 16:17:35 2017 +0200 ---------------------------------------------------------------------- .../apache/beam/runners/spark/translation/BoundedDataset.java | 6 ++---- 1 file changed, 2 insertions(+), 4 deletions(-) ---------------------------------------------------------------------- http://git-wip-us.apache.org/repos/asf/beam/blob/4febd954/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/BoundedDataset.java ---------------------------------------------------------------------- diff --git a/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/BoundedDataset.java b/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/BoundedDataset.java index 5e19846..7db04a8 100644 --- a/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/BoundedDataset.java +++ b/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/BoundedDataset.java @@ -20,7 +20,6 @@ package org.apache.beam.runners.spark.translation; import com.google.common.base.Function; import com.google.common.collect.Iterables; -import java.util.Iterator; import java.util.List; import javax.annotation.Nullable; import org.apache.beam.runners.spark.coders.CoderHelpers; @@ -107,9 +106,8 @@ public class BoundedDataset implements Dataset { @Override public void action() { // Empty function to force computation of RDD. - rdd.foreachPartition(new VoidFunction>>() { - @Override - public void call(Iterator> windowedValueIterator) throws Exception { + rdd.foreach(new VoidFunction>() { + @Override public void call(WindowedValue tWindowedValue) throws Exception { // Empty implementation. } });