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

    https://github.com/apache/flink/pull/1929#discussion_r61231656
  
    --- Diff: 
flink-streaming-java/src/main/java/org/apache/flink/streaming/api/functions/source/FileSplitReadOperator.java
 ---
    @@ -0,0 +1,176 @@
    +/*
    + * 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.flink.streaming.api.functions.source;
    +
    +import org.apache.flink.api.common.io.FileInputFormat;
    +import org.apache.flink.api.common.typeinfo.TypeInformation;
    +import org.apache.flink.api.common.typeutils.TypeSerializer;
    +import org.apache.flink.configuration.Configuration;
    +import org.apache.flink.core.fs.FileInputSplit;
    +import org.apache.flink.streaming.api.operators.AbstractStreamOperator;
    +import org.apache.flink.streaming.api.operators.ChainingStrategy;
    +import org.apache.flink.streaming.api.operators.OneInputStreamOperator;
    +import org.apache.flink.streaming.api.operators.TimestampedCollector;
    +import org.apache.flink.streaming.api.watermark.Watermark;
    +import org.apache.flink.streaming.runtime.streamrecord.StreamRecord;
    +import org.slf4j.Logger;
    +import org.slf4j.LoggerFactory;
    +
    +import java.io.IOException;
    +import java.util.Queue;
    +import java.util.concurrent.ConcurrentLinkedQueue;
    +
    +import static org.apache.flink.util.Preconditions.checkNotNull;
    +
    +/**
    + * This is the operator that reads the splits received from {@link 
FileSplitMonitoringFunction}.
    + * This operator will receive just the split descriptors and then read and 
emit records. This may lead
    + * to backpressure. To avoid this, we will have another thread actually 
reading the splits and
    + * another forwarding the checkpoint barriers. The two should sync so that 
the checkpoints reflect the
    + * current state.
    + * */
    +public class FileSplitReadOperator<OUT> extends 
AbstractStreamOperator<OUT> implements OneInputStreamOperator<FileInputSplit, 
OUT> {
    +
    +   private static final Logger LOG = 
LoggerFactory.getLogger(FileSplitMonitoringFunction.class);
    +
    +   private Queue<FileInputSplit> unReadSplits;
    +
    +   private transient SplitReader reader;
    +   private transient Object checkpointLock;
    +   private transient TimestampedCollector<OUT> collector;
    +
    +   private Configuration configuration;
    +   private FileInputFormat<OUT> format;
    +   private TypeInformation<OUT> typeInfo;
    +   private transient TypeSerializer<OUT> serializer;
    +
    +   public FileSplitReadOperator(FileInputFormat<OUT> format, 
TypeInformation<OUT> typeInfo, Configuration configuration) {
    +           this.format = format;
    +           this.typeInfo = typeInfo;
    +           this.configuration = configuration;
    +           this.unReadSplits = new ConcurrentLinkedQueue<>();
    +
    +           // this is for the extra thread that is reading,
    +           // the tests were not terminating because the success
    +           // exception was caught by the extra thread, and never
    +           // forwarded.
    +
    +           // for now, and to keep the extra thread to avoid backpressure,
    +           // we just disable chaining for the operator, so that it runs on
    +           // another thread. This will change later for a cleaner design.
    +           setChainingStrategy(ChainingStrategy.NEVER);
    +   }
    +
    +   @Override
    +   public void open() throws Exception {
    +           super.open();
    +
    +           this.format.configure(configuration);
    +           this.collector = new TimestampedCollector<OUT>(output);
    +           this.checkpointLock = getContainingTask()
    +                   .getCheckpointLock();
    +           this.serializer = 
typeInfo.createSerializer(getRuntimeContext().getExecutionConfig());
    +
    +           this.reader = new SplitReader(unReadSplits, format, serializer, 
collector, checkpointLock);
    +           this.reader.start();
    +   }
    +
    +   @Override
    +   public void processElement(StreamRecord<FileInputSplit> element) throws 
Exception {
    +           LOG.info("Reading Split: " + element.getValue());
    +           this.unReadSplits.add(element.getValue());
    +   }
    +
    +   @Override
    +   public void processWatermark(Watermark mark) throws Exception {
    +           output.emitWatermark(mark);
    +   }
    +
    +   @Override
    +   public void close() throws Exception {
    +           super.close();
    +           this.reader.cancel();
    +           this.collector.close();
    +           this.format.close();
    --- End diff --
    
    I think `InputFormat.close()` is intended as the opposite of `open()`, 
which is for input splits. Therefore, it should be called in the reader after 
finishing reading of a split. Protected by a `finally` just to make sure that 
it is always called.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastruct...@apache.org or file a JIRA ticket
with INFRA.
---

Reply via email to