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

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

                Author: ASF GitHub Bot
            Created on: 03/Nov/20 13:52
            Start Date: 03/Nov/20 13:52
    Worklog Time Spent: 10m 
      Work Description: mxm commented on a change in pull request #13240:
URL: https://github.com/apache/beam/pull/13240#discussion_r516185424



##########
File path: 
runners/flink/src/main/java/org/apache/beam/runners/flink/translation/wrappers/streaming/DoFnOperator.java
##########
@@ -1194,29 +1200,35 @@ public void verifyDeterministic() throws 
NonDeterministicException {
     private Map<TupleTag<?>, Integer> tagsToIds;
     private Map<TupleTag<?>, OutputTag<WindowedValue<?>>> tagsToOutputTags;
     private Map<TupleTag<?>, Coder<WindowedValue<?>>> tagsToCoders;
+    private SerializablePipelineOptions pipelineOptions;

Review comment:
       ```suggestion
       private final SerializablePipelineOptions pipelineOptions;
   ```

##########
File path: 
runners/flink/src/test/java/org/apache/beam/runners/flink/translation/wrappers/streaming/DoFnOperatorTest.java
##########
@@ -761,7 +797,9 @@ void testSideInputs(boolean keyed) throws Exception {
               doFnOperator,
               keySelector,
               null,
-              new CoderTypeInformation<>(FlinkKeyUtils.ByteBufferCoder.of()));
+              new CoderTypeInformation<>(
+                  FlinkKeyUtils.ByteBufferCoder.of(),
+                  PipelineOptionsFactory.as(FlinkPipelineOptions.class)));

Review comment:
       Perhaps, create a static factory method for all of these defaults? E.g. 
`FlinkPipelineOptions.default()`

##########
File path: 
runners/flink/1.8/src/main/java/org/apache/beam/runners/flink/translation/types/CoderTypeSerializer.java
##########
@@ -82,10 +81,14 @@ public T createInstance() {
 
   @Override
   public T copy(T t) {

Review comment:
       This is a Flink interface method which we can't rename.

##########
File path: 
runners/flink/1.8/src/main/java/org/apache/beam/runners/flink/translation/types/CoderTypeSerializer.java
##########
@@ -49,20 +50,18 @@
    * org.apache.beam.sdk.io.FileSystems} registration needed for {@link
    * org.apache.beam.sdk.transforms.Reshuffle} translation.
    */
-  @SuppressWarnings("unused")
-  private final @Nullable SerializablePipelineOptions pipelineOptions;
+  private final SerializablePipelineOptions pipelineOptions;
 
-  public CoderTypeSerializer(Coder<T> coder) {
-    Preconditions.checkNotNull(coder);
-    this.coder = coder;
-    this.pipelineOptions = null;
-  }
+  private final boolean fasterCopy;

Review comment:
       Folks, this is an internal boolean flag which is only used at a single 
place. If in doubt, one can clearly see what it does. We could add a `/** */` 
Javadoc field comment for it if you will. IMHO the name is really personal 
preference and I have nothing against `fasterCopy` as it clearly indicates that 
the `copy()` method will return faster. Nothing semantically wrong about that. 
`zeroCopy` is not 100% true because we will perform a copy for primitive types 
but please let's not argue about that. `DisableExcessCopy` is too opinionated 
IMHO 😇 




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to 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: 506853)
    Time Spent: 40m  (was: 0.5h)

> Add option to disable copying between Flink runner 
> ---------------------------------------------------
>
>                 Key: BEAM-11146
>                 URL: https://issues.apache.org/jira/browse/BEAM-11146
>             Project: Beam
>          Issue Type: Improvement
>          Components: runner-flink
>            Reporter: Teodor Spæren
>            Assignee: Teodor Spæren
>            Priority: P2
>              Labels: performance
>          Time Spent: 40m
>  Remaining Estimate: 0h
>
> In order to implement Flink 
> [TypeSerializer|https://github.com/apache/flink/blob/master/flink-core/src/main/java/org/apache/flink/api/common/typeutils/TypeSerializer.java]
>  the runner implements 
> [CoderTypeSerializer|https://github.com/apache/beam/blob/master/runners/flink/1.8/src/main/java/org/apache/beam/runners/flink/translation/types/CoderTypeSerializer.java#L84].
>  The way the {{copy}} function is implemented is by first serializing and 
> then deserializing each element. This means that such a deep copy needs to be 
> done between each operator and this can become a bottleneck.
> The reason the {{copy}} functions need to be implemented is that Flink 
> guarantees that elements will be deep copied between each operator. In Beam 
> this is the users responsibility and so this is not strictly neccecarry.
> The aim of this improvement is to introduce an option on the Flink Runner, 
> that eliminates this overhead, by simply returning the value.
> [Here is the mailing list 
> discussion|https://lists.apache.org/thread.html/r24129dba98782e1cf4d18ec738ab9714dceb05ac23f13adfac5baad1%40%3Cdev.beam.apache.org%3E]



--
This message was sent by Atlassian Jira
(v8.3.4#803005)

Reply via email to