[ 
https://issues.apache.org/jira/browse/BEAM-4285?focusedWorklogId=113407&page=com.atlassian.jira.plugin.system.issuetabpanels:worklog-tabpanel#worklog-113407
 ]

ASF GitHub Bot logged work on BEAM-4285:
----------------------------------------

                Author: ASF GitHub Bot
            Created on: 19/Jun/18 22:14
            Start Date: 19/Jun/18 22:14
    Worklog Time Spent: 10m 
      Work Description: jkff commented on a change in pull request #5688: 
[BEAM-4285] Implement Flink batch side input handler
URL: https://github.com/apache/beam/pull/5688#discussion_r196595596
 
 

 ##########
 File path: 
runners/flink/src/main/java/org/apache/beam/runners/flink/translation/functions/FlinkBatchSideInputHandlerFactory.java
 ##########
 @@ -0,0 +1,131 @@
+/*
+ * 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.runners.flink.translation.functions;
+
+import static com.google.common.base.Preconditions.checkArgument;
+
+import com.google.auto.value.AutoValue;
+import com.google.common.collect.ImmutableMap;
+import com.google.common.collect.ImmutableMultimap;
+import com.google.common.collect.Multimap;
+import java.util.List;
+import java.util.Map;
+import javax.annotation.Nullable;
+import 
org.apache.beam.model.pipeline.v1.RunnerApi.ExecutableStagePayload.SideInputId;
+import org.apache.beam.runners.core.construction.graph.ExecutableStage;
+import 
org.apache.beam.runners.core.construction.graph.PipelineNode.PCollectionNode;
+import org.apache.beam.runners.core.construction.graph.SideInputReference;
+import org.apache.beam.runners.fnexecution.state.StateRequestHandler;
+import 
org.apache.beam.runners.fnexecution.state.StateRequestHandlers.MultimapSideInputHandler;
+import 
org.apache.beam.runners.fnexecution.state.StateRequestHandlers.MultimapSideInputHandlerFactory;
+import org.apache.beam.sdk.coders.Coder;
+import org.apache.beam.sdk.transforms.windowing.BoundedWindow;
+import org.apache.beam.sdk.util.WindowedValue;
+import org.apache.beam.sdk.values.KV;
+import org.apache.flink.api.common.functions.RuntimeContext;
+
+/**
+ * {@link StateRequestHandler} that uses a Flink {@link RuntimeContext} to 
access Flink broadcast
+ * variable that represent side inputs.
+ */
+class FlinkBatchSideInputHandlerFactory implements 
MultimapSideInputHandlerFactory {
+
+  // Map from side input id to global PCollection id.
+  private final Map<SideInputId, PCollectionNode> sideInputToCollection;
+  private final RuntimeContext runtimeContext;
+
+  /**
+   * Creates a new state handler for the given stage. Note that this requires 
a traversal of the
+   * stage itself, so this should only be called once per stage rather than 
once per bundle.
+   */
+  static FlinkBatchSideInputHandlerFactory forStage(
+      ExecutableStage stage, RuntimeContext runtimeContext) {
+    ImmutableMap.Builder<SideInputId, PCollectionNode> sideInputBuilder = 
ImmutableMap.builder();
+    for (SideInputReference sideInput : stage.getSideInputs()) {
+      sideInputBuilder.put(
+          SideInputId.newBuilder()
+              .setTransformId(sideInput.transform().getId())
+              .setLocalName(sideInput.localName())
+              .build(),
+          sideInput.collection());
+    }
+    return new FlinkBatchSideInputHandlerFactory(sideInputBuilder.build(), 
runtimeContext);
+  }
+
+  private FlinkBatchSideInputHandlerFactory(
+      Map<SideInputId, PCollectionNode> sideInputToCollection, RuntimeContext 
runtimeContext) {
+    this.sideInputToCollection = sideInputToCollection;
+    this.runtimeContext = runtimeContext;
+  }
+
+  @Override
+  public <K, V, W extends BoundedWindow> MultimapSideInputHandler<K, V, W> 
forSideInput(
+      String transformId,
+      String sideInputId,
+      Coder<K> keyCoder,
+      Coder<V> valueCoder,
+      Coder<W> windowCoder) {
+    PCollectionNode collectionNode =
+        sideInputToCollection.get(
+            
SideInputId.newBuilder().setTransformId(transformId).setLocalName(sideInputId).build());
+    checkArgument(collectionNode != null, "No side input for %s/%s", 
transformId, sideInputId);
+    List<Object> broadcastVariable = 
runtimeContext.getBroadcastVariable(collectionNode.getId());
+
+    ImmutableMultimap.Builder<SideInputKey<K, W>, V> multimap = 
ImmutableMultimap.builder();
+    for (Object element : broadcastVariable) {
+      WindowedValue<KV<K, V>> windowedValue = (WindowedValue<KV<K, V>>) 
element;
+      K key = windowedValue.getValue().getKey();
+      V value = windowedValue.getValue().getValue();
+
+      for (BoundedWindow boundedWindow : windowedValue.getWindows()) {
+        @SuppressWarnings("unchecked")
+        W window = (W) boundedWindow;
+        multimap.put(SideInputKey.of(key, window), value);
+      }
+    }
+
+    return new SideInputHandler<>(multimap.build());
+  }
+
+  private static class SideInputHandler<K, V, W extends BoundedWindow>
+      implements MultimapSideInputHandler<K, V, W> {
+
+    private final Multimap<SideInputKey<K, W>, V> collection;
+
+    private SideInputHandler(Multimap<SideInputKey<K, W>, V> collection) {
+      this.collection = collection;
+    }
+
+    @Override
+    public Iterable<V> get(K key, W window) {
 
 Review comment:
   Just to clarify, is this the main input window or the side input window? (if 
it's the latter then it's fine)

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
[email protected]


Issue Time Tracking
-------------------

    Worklog Id:     (was: 113407)
    Time Spent: 50m  (was: 40m)

> Flink batch state request handler
> ---------------------------------
>
>                 Key: BEAM-4285
>                 URL: https://issues.apache.org/jira/browse/BEAM-4285
>             Project: Beam
>          Issue Type: Bug
>          Components: runner-flink
>            Reporter: Ben Sidhom
>            Assignee: Ben Sidhom
>            Priority: Major
>          Time Spent: 50m
>  Remaining Estimate: 0h
>
> In order to support side inputs Flink needs a state service request handler. 
> As in the non-portable we can start by handling batch side inputs by Flink 
> broadcast variables.
> [https://github.com/bsidhom/beam/blob/41de3bce60f1ebc9211f299612a20d8e561f9b6f/runners/flink/src/main/java/org/apache/beam/runners/flink/translation/functions/FlinkBatchStateRequestHandler.java]
>  or 
> [https://github.com/bsidhom/beam/blob/41de3bce60f1ebc9211f299612a20d8e561f9b6f/runners/flink/src/main/java/org/apache/beam/runners/flink/translation/functions/FlinkBatchStateRequestHandler.java]
>  can be used as a starting point. 



--
This message was sent by Atlassian JIRA
(v7.6.3#76005)

Reply via email to