You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@beam.apache.org by lc...@apache.org on 2016/07/06 17:20:13 UTC

[07/50] [abbrv] incubator-beam git commit: Add StaticWindows

Add StaticWindows

This is a windowFn that ignores the input and always assigns to the same
input. It returns the provided window as the side input window if and
only if that window is present within its set of windows.


Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo
Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/f146f06f
Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/f146f06f
Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/f146f06f

Branch: refs/heads/runners-spark2
Commit: f146f06f9fb9a5d100a322b2747ccc89f13c70d3
Parents: ebea5a7
Author: Thomas Groh <tg...@google.com>
Authored: Tue Jun 21 10:44:43 2016 -0700
Committer: Luke Cwik <lc...@google.com>
Committed: Wed Jul 6 10:18:49 2016 -0700

----------------------------------------------------------------------
 .../apache/beam/sdk/testing/StaticWindows.java  | 110 +++++++++++++++++++
 .../apache/beam/sdk/testing/WindowSupplier.java |  83 ++++++++++++++
 .../beam/sdk/testing/StaticWindowsTest.java     |  94 ++++++++++++++++
 .../beam/sdk/testing/WindowSupplierTest.java    |  89 +++++++++++++++
 4 files changed, 376 insertions(+)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/f146f06f/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/StaticWindows.java
----------------------------------------------------------------------
diff --git a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/StaticWindows.java b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/StaticWindows.java
new file mode 100644
index 0000000..08d2355
--- /dev/null
+++ b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/StaticWindows.java
@@ -0,0 +1,110 @@
+/*
+ * 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 static com.google.common.base.Preconditions.checkArgument;
+
+import org.apache.beam.sdk.coders.Coder;
+import org.apache.beam.sdk.transforms.windowing.BoundedWindow;
+import org.apache.beam.sdk.transforms.windowing.NonMergingWindowFn;
+import org.apache.beam.sdk.transforms.windowing.WindowFn;
+
+import com.google.common.base.Supplier;
+import com.google.common.collect.Iterables;
+
+import java.util.Collection;
+import java.util.Collections;
+import java.util.Objects;
+
+/**
+ * A {@link WindowFn} that assigns all elements to a static collection of
+ * {@link BoundedWindow BoundedWindows}. Side inputs windowed into static windows only support
+ * main input windows in the provided collection of windows.
+ */
+final class StaticWindows extends NonMergingWindowFn<Object, BoundedWindow> {
+  private final Supplier<Collection<BoundedWindow>> windows;
+  private final Coder<BoundedWindow> coder;
+
+  private final boolean onlyExisting;
+
+  private StaticWindows(
+      Supplier<Collection<BoundedWindow>> windows,
+      Coder<BoundedWindow> coder,
+      boolean onlyExisting) {
+    this.windows = windows;
+    this.coder = coder;
+    this.onlyExisting = onlyExisting;
+  }
+
+  public static <W extends BoundedWindow> StaticWindows of(Coder<W> coder, Iterable<W> windows) {
+    checkArgument(!Iterables.isEmpty(windows), "Input windows to StaticWindows may not be empty");
+    @SuppressWarnings("unchecked")
+    StaticWindows windowFn =
+        new StaticWindows(
+            WindowSupplier.of((Coder<BoundedWindow>) coder, (Iterable<BoundedWindow>) windows),
+            (Coder<BoundedWindow>) coder,
+            false);
+    return windowFn;
+  }
+
+  public static <W extends BoundedWindow> StaticWindows of(Coder<W> coder, W window) {
+    return of(coder, Collections.singleton(window));
+  }
+
+  public StaticWindows intoOnlyExisting() {
+    return new StaticWindows(windows, coder, true);
+  }
+
+  public Collection<BoundedWindow> getWindows() {
+    return windows.get();
+  }
+
+  @Override
+  public Collection<BoundedWindow> assignWindows(AssignContext c) throws Exception {
+    if (onlyExisting) {
+      checkArgument(
+          windows.get().contains(c.window()),
+          "Tried to assign windows to an element that is not already windowed into a provided "
+              + "window when onlyExisting is set to true");
+      return Collections.singleton(c.window());
+    } else {
+      return getWindows();
+    }
+  }
+
+  @Override
+  public boolean isCompatible(WindowFn<?, ?> other) {
+    if (!(other instanceof StaticWindows)) {
+      return false;
+    }
+    StaticWindows that = (StaticWindows) other;
+    return Objects.equals(this.windows.get(), that.windows.get());
+  }
+
+  @Override
+  public Coder<BoundedWindow> windowCoder() {
+    return coder;
+  }
+
+  @Override
+  public BoundedWindow getSideInputWindow(BoundedWindow window) {
+    checkArgument(windows.get().contains(window),
+        "StaticWindows only supports side input windows for main input windows that it contains");
+    return window;
+  }
+}

http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/f146f06f/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/WindowSupplier.java
----------------------------------------------------------------------
diff --git a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/WindowSupplier.java b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/WindowSupplier.java
new file mode 100644
index 0000000..62bc09f
--- /dev/null
+++ b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/WindowSupplier.java
@@ -0,0 +1,83 @@
+/*
+ * 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 org.apache.beam.sdk.coders.Coder;
+import org.apache.beam.sdk.coders.CoderException;
+import org.apache.beam.sdk.transforms.windowing.BoundedWindow;
+import org.apache.beam.sdk.util.CoderUtils;
+
+import com.google.common.base.Supplier;
+import com.google.common.collect.ImmutableList;
+import com.google.common.collect.ImmutableSet;
+
+import java.io.Serializable;
+import java.util.Collection;
+
+/**
+ * A {@link Supplier} that returns a static set of {@link BoundedWindow BoundedWindows}. The
+ * supplier is {@link Serializable}, and handles encoding and decoding the windows with a
+ * {@link Coder} provided for the windows.
+ */
+final class WindowSupplier implements Supplier<Collection<BoundedWindow>>, Serializable {
+  private final Coder<? extends BoundedWindow> coder;
+  private final Collection<byte[]> encodedWindows;
+
+  private transient Collection<BoundedWindow> windows;
+
+  public static <W extends BoundedWindow> WindowSupplier of(Coder<W> coder, Iterable<W> windows) {
+    ImmutableSet.Builder<byte[]> windowsBuilder = ImmutableSet.builder();
+    for (W window : windows) {
+      try {
+        windowsBuilder.add(CoderUtils.encodeToByteArray(coder, window));
+      } catch (CoderException e) {
+        throw new IllegalArgumentException(
+            "Could not encode provided windows with the provided window coder", e);
+      }
+    }
+    return new WindowSupplier(coder, windowsBuilder.build());
+  }
+
+  private WindowSupplier(Coder<? extends BoundedWindow> coder, Collection<byte[]> encodedWindows) {
+    this.coder = coder;
+    this.encodedWindows = encodedWindows;
+  }
+
+  @Override
+  public Collection<BoundedWindow> get() {
+    if (windows == null) {
+      decodeWindows();
+    }
+    return windows;
+  }
+
+  private synchronized void decodeWindows() {
+    if (windows == null) {
+      ImmutableList.Builder<BoundedWindow> windowsBuilder = ImmutableList.builder();
+      for (byte[] encoded : encodedWindows) {
+        try {
+          windowsBuilder.add(CoderUtils.decodeFromByteArray(coder, encoded));
+        } catch (CoderException e) {
+          throw new IllegalArgumentException(
+              "Could not decode provided windows with the provided window coder", e);
+        }
+      }
+      this.windows = windowsBuilder.build();
+    }
+  }
+}

http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/f146f06f/sdks/java/core/src/test/java/org/apache/beam/sdk/testing/StaticWindowsTest.java
----------------------------------------------------------------------
diff --git a/sdks/java/core/src/test/java/org/apache/beam/sdk/testing/StaticWindowsTest.java b/sdks/java/core/src/test/java/org/apache/beam/sdk/testing/StaticWindowsTest.java
new file mode 100644
index 0000000..fd715dc
--- /dev/null
+++ b/sdks/java/core/src/test/java/org/apache/beam/sdk/testing/StaticWindowsTest.java
@@ -0,0 +1,94 @@
+/*
+ * 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 static org.junit.Assert.assertThat;
+
+import org.apache.beam.sdk.transforms.windowing.BoundedWindow;
+import org.apache.beam.sdk.transforms.windowing.GlobalWindow;
+import org.apache.beam.sdk.transforms.windowing.IntervalWindow;
+import org.apache.beam.sdk.transforms.windowing.WindowFn;
+
+import com.google.common.collect.ImmutableList;
+
+import org.hamcrest.Matchers;
+import org.joda.time.Instant;
+import org.junit.Rule;
+import org.junit.Test;
+import org.junit.rules.ExpectedException;
+import org.junit.runner.RunWith;
+import org.junit.runners.JUnit4;
+
+/**
+ * Tests for {@link StaticWindows}.
+ */
+@RunWith(JUnit4.class)
+public class StaticWindowsTest {
+  @Rule
+  public ExpectedException thrown = ExpectedException.none();
+
+  private final IntervalWindow first = new IntervalWindow(new Instant(0), new Instant(100_000L));
+  private final IntervalWindow second =
+      new IntervalWindow(new Instant(1_000_000L), GlobalWindow.INSTANCE.maxTimestamp());
+
+  @Test
+  public void singleWindowSucceeds() throws Exception {
+    WindowFn<Object, BoundedWindow> fn = StaticWindows.of(IntervalWindow.getCoder(), first);
+    assertThat(WindowFnTestUtils.assignedWindows(fn, 100L),
+        Matchers.<BoundedWindow>contains(first));
+    assertThat(WindowFnTestUtils.assignedWindows(fn, -100L),
+        Matchers.<BoundedWindow>contains(first));
+  }
+
+  @Test
+  public void multipleWindowsSucceeds() throws Exception {
+    WindowFn<Object, BoundedWindow> fn =
+        StaticWindows.of(IntervalWindow.getCoder(), ImmutableList.of(first, second));
+    assertThat(WindowFnTestUtils.assignedWindows(fn, 100L),
+        Matchers.<BoundedWindow>containsInAnyOrder(first, second));
+    assertThat(WindowFnTestUtils.assignedWindows(fn, 1_000_000_000L),
+        Matchers.<BoundedWindow>containsInAnyOrder(first, second));
+    assertThat(WindowFnTestUtils.assignedWindows(fn, -100L),
+        Matchers.<BoundedWindow>containsInAnyOrder(first, second));
+  }
+
+  @Test
+  public void getSideInputWindowIdentity() {
+    WindowFn<Object, BoundedWindow> fn =
+        StaticWindows.of(IntervalWindow.getCoder(), ImmutableList.of(first, second));
+
+    assertThat(fn.getSideInputWindow(first), Matchers.<BoundedWindow>equalTo(first));
+    assertThat(fn.getSideInputWindow(second), Matchers.<BoundedWindow>equalTo(second));
+  }
+
+  @Test
+  public void getSideInputWindowNotPresent() {
+    WindowFn<Object, BoundedWindow> fn =
+        StaticWindows.of(IntervalWindow.getCoder(), ImmutableList.of(second));
+    thrown.expect(IllegalArgumentException.class);
+    thrown.expectMessage("contains");
+    fn.getSideInputWindow(first);
+  }
+
+  @Test
+  public void emptyIterableThrows() {
+    thrown.expect(IllegalArgumentException.class);
+    thrown.expectMessage("may not be empty");
+    StaticWindows.of(GlobalWindow.Coder.INSTANCE, ImmutableList.<GlobalWindow>of());
+  }
+}

http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/f146f06f/sdks/java/core/src/test/java/org/apache/beam/sdk/testing/WindowSupplierTest.java
----------------------------------------------------------------------
diff --git a/sdks/java/core/src/test/java/org/apache/beam/sdk/testing/WindowSupplierTest.java b/sdks/java/core/src/test/java/org/apache/beam/sdk/testing/WindowSupplierTest.java
new file mode 100644
index 0000000..178c67c
--- /dev/null
+++ b/sdks/java/core/src/test/java/org/apache/beam/sdk/testing/WindowSupplierTest.java
@@ -0,0 +1,89 @@
+/*
+ * 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 static org.junit.Assert.assertThat;
+
+import org.apache.beam.sdk.coders.AtomicCoder;
+import org.apache.beam.sdk.coders.CoderException;
+import org.apache.beam.sdk.transforms.windowing.BoundedWindow;
+import org.apache.beam.sdk.transforms.windowing.IntervalWindow;
+import org.apache.beam.sdk.util.SerializableUtils;
+
+import com.google.common.collect.ImmutableList;
+
+import org.hamcrest.Matchers;
+import org.joda.time.Instant;
+import org.junit.Rule;
+import org.junit.Test;
+import org.junit.rules.ExpectedException;
+
+import java.io.IOException;
+import java.io.InputStream;
+import java.io.OutputStream;
+import java.util.Collections;
+
+/**
+ * Tests for {@link WindowSupplier}.
+ */
+public class WindowSupplierTest {
+  private final IntervalWindow window = new IntervalWindow(new Instant(0L), new Instant(100L));
+  private final IntervalWindow otherWindow =
+      new IntervalWindow(new Instant(-100L), new Instant(100L));
+  @Rule public ExpectedException thrown = ExpectedException.none();
+
+  @Test
+  public void getReturnsProvidedWindows() {
+    assertThat(
+        WindowSupplier.of(IntervalWindow.getCoder(), ImmutableList.of(window, otherWindow)).get(),
+        Matchers.<BoundedWindow>containsInAnyOrder(otherWindow, window));
+  }
+
+  @Test
+  public void getAfterSerialization() {
+    WindowSupplier supplier =
+        WindowSupplier.of(IntervalWindow.getCoder(), ImmutableList.of(window, otherWindow));
+    assertThat(
+        SerializableUtils.clone(supplier).get(),
+        Matchers.<BoundedWindow>containsInAnyOrder(otherWindow, window));
+  }
+
+  @Test
+  public void unencodableWindowFails() {
+    thrown.expect(IllegalArgumentException.class);
+    thrown.expectMessage("Could not encode");
+    WindowSupplier.of(
+        new FailingCoder(),
+        Collections.<BoundedWindow>singleton(window));
+  }
+
+  private static class FailingCoder extends AtomicCoder<BoundedWindow>  {
+    @Override
+    public void encode(
+        BoundedWindow value, OutputStream outStream, Context context)
+        throws CoderException, IOException {
+      throw new CoderException("Test Enccode Exception");
+    }
+
+    @Override
+    public BoundedWindow decode(
+        InputStream inStream, Context context) throws CoderException, IOException {
+      throw new CoderException("Test Decode Exception");
+    }
+  }
+}