[ 
https://issues.apache.org/jira/browse/FLINK-2124?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=14595545#comment-14595545
 ] 

ASF GitHub Bot commented on FLINK-2124:
---------------------------------------

Github user jreiffers commented on a diff in the pull request:

    https://github.com/apache/flink/pull/848#discussion_r32916185
  
    --- Diff: 
flink-staging/flink-streaming/flink-streaming-core/src/main/java/org/apache/flink/streaming/api/functions/source/FromElementsFunction.java
 ---
    @@ -17,37 +17,81 @@
     
     package org.apache.flink.streaming.api.functions.source;
     
    -import java.util.Arrays;
    -import java.util.Collection;
    +import org.apache.flink.api.common.typeutils.TypeSerializer;
    +import org.apache.flink.core.memory.DataInputView;
    +import org.apache.flink.core.memory.InputViewDataInputStreamWrapper;
    +import org.apache.flink.core.memory.OutputViewDataOutputStreamWrapper;
    +
    +import java.io.ByteArrayInputStream;
    +import java.io.ByteArrayOutputStream;
    +import java.io.DataInputStream;
    +import java.io.DataOutputStream;
    +import java.io.IOException;
     import java.util.Iterator;
     
     public class FromElementsFunction<T> implements SourceFunction<T> {
        
        private static final long serialVersionUID = 1L;
     
    -   private Iterable<T> iterable;
    +   private final TypeSerializer<T> serializer;
    +   private final byte[] elements;
     
        private volatile boolean isRunning = true;
     
    -   public FromElementsFunction(T... elements) {
    -           this.iterable = Arrays.asList(elements);
    -   }
    +   public FromElementsFunction(TypeSerializer<T> serializer, final T... 
elements) {
    +           this(serializer, new Iterable<T>() {
    --- End diff --
    
    Maybe. Arrays.asList also unnecessarily copies the array.


> FromElementsFunction is not really Serializable
> -----------------------------------------------
>
>                 Key: FLINK-2124
>                 URL: https://issues.apache.org/jira/browse/FLINK-2124
>             Project: Flink
>          Issue Type: Bug
>          Components: Streaming
>            Reporter: Aljoscha Krettek
>            Assignee: Johannes Reifferscheid
>
> The function stores an Iterable of T. T is not necessarily Serializable and 
> and Iterable is also not necessarily Serializable.



--
This message was sent by Atlassian JIRA
(v6.3.4#6332)

Reply via email to