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

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

                Author: ASF GitHub Bot
            Created on: 25/Feb/20 14:37
            Start Date: 25/Feb/20 14:37
    Worklog Time Spent: 10m 
      Work Description: sunjincheng121 commented on issue #10945: [BEAM-9295] 
Add Flink 1.10 build target and Make FlinkRunner compatible with Flink 1.10
URL: https://github.com/apache/beam/pull/10945#issuecomment-590897942
 
 
   The test case `test_large_elements` failed with execption:
   ```
   java.io.IOException: Cannot write record to fresh sort buffer. Record too 
large.
           at 
org.apache.flink.runtime.operators.chaining.SynchronousChainedCombineDriver.collect(SynchronousChainedCombineDriver.java:176)
           at 
org.apache.flink.runtime.operators.util.metrics.CountingCollector.collect(CountingCollector.java:35)
           at 
org.apache.flink.runtime.operators.MapDriver.run(MapDriver.java:103)
           at 
org.apache.flink.runtime.operators.BatchTask.run(BatchTask.java:504)
           at 
org.apache.flink.runtime.operators.BatchTask.invoke(BatchTask.java:369)
           at org.apache.flink.runtime.taskmanager.Task.doRun(Task.java:707)
           at org.apache.flink.runtime.taskmanager.Task.run(Task.java:532)  
   ```
   
   I guess it's related to the feature "Unified Memory Configuration for 
TaskExecutors"(https://issues.apache.org/jira/browse/FLINK-13980) which is 
introduced in Flink 1.10. Before 1.10, the memory managed by the Flink's 
MemoryManager is calculated dynamically if not configured and I have checked 
that it will be about 2500 MB in my local machine. Since 1.10, it will be 128 
MB if not configured (taskmanager.memory.managed.size).
   
   I have performed a simple test and the failed test `test_large_elements` 
could pass after adding the following code at 
https://github.com/apache/beam/blob/7b3a3fa6c9291692b56dbc358dfc075724b993b6/runners/flink/src/main/java/org/apache/beam/runners/flink/FlinkExecutionEnvironments.java#L77
 :
   
   ```
   flinkConfiguration.set(TaskManagerOptions.MANAGED_MEMORY_SIZE, 
MemorySize.parse("2048m"));
   ```
   
   I'm still investigating the best way to address this issue at Beam side.
 
----------------------------------------------------------------
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: 392584)
    Time Spent: 1h  (was: 50m)

> Add Flink 1.10 build target and Make FlinkRunner compatible with Flink 1.10
> ---------------------------------------------------------------------------
>
>                 Key: BEAM-9295
>                 URL: https://issues.apache.org/jira/browse/BEAM-9295
>             Project: Beam
>          Issue Type: New Feature
>          Components: runner-flink
>            Reporter: sunjincheng
>            Assignee: sunjincheng
>            Priority: Major
>          Time Spent: 1h
>  Remaining Estimate: 0h
>
> Apache Flink 1.10 has completed the final release vote, see [1]. So, I would 
> like to add Flink 1.10 build target and make Flink Runner compatible with 
> Flink 1.10.
> And I appreciate it if you can leave your suggestions or comments!
> [1] 
> https://lists.apache.org/thread.html/r97672d4d1e47372cebf23e6643a6cc30a06bfbdf3f277b0be3695b15%40%3Cdev.flink.apache.org%3E



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

Reply via email to