yunfengzhou-hub commented on code in PR #97: URL: https://github.com/apache/flink-ml/pull/97#discussion_r889667003
########## flink-ml-iteration/src/main/java/org/apache/flink/iteration/datacache/nonkeyed/DataCache.java: ########## @@ -0,0 +1,351 @@ +/* + * 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.iteration.datacache.nonkeyed; + +import org.apache.flink.annotation.Internal; +import org.apache.flink.api.common.typeutils.TypeSerializer; +import org.apache.flink.core.fs.FSDataInputStream; +import org.apache.flink.core.fs.FSDataOutputStream; +import org.apache.flink.core.fs.FileSystem; +import org.apache.flink.core.fs.Path; +import org.apache.flink.core.memory.DataInputViewStreamWrapper; +import org.apache.flink.runtime.util.NonClosingInputStreamDecorator; +import org.apache.flink.runtime.util.NonClosingOutputStreamDecorator; +import org.apache.flink.statefun.flink.core.feedback.FeedbackConsumer; +import org.apache.flink.table.runtime.util.MemorySegmentPool; +import org.apache.flink.util.IOUtils; +import org.apache.flink.util.function.SupplierWithException; + +import org.apache.commons.io.input.BoundedInputStream; + +import java.io.DataInputStream; +import java.io.DataOutputStream; +import java.io.IOException; +import java.io.InputStream; +import java.io.OutputStream; +import java.util.ArrayList; +import java.util.Collections; +import java.util.List; + +import static org.apache.flink.util.Preconditions.checkState; + +/** Records the data received and replays them on required. */ +@Internal +public class DataCache<T> implements Iterable<T> { + + private static final int CURRENT_VERSION = 1; + + private final TypeSerializer<T> serializer; + + private final FileSystem fileSystem; + + private final SupplierWithException<Path, IOException> pathGenerator; + + private final MemorySegmentPool segmentPool; + + private final List<Segment> finishedSegments; + + private SegmentWriter<T> currentWriter; + + public DataCache( + TypeSerializer<T> serializer, + FileSystem fileSystem, + SupplierWithException<Path, IOException> pathGenerator) + throws IOException { + this(serializer, fileSystem, pathGenerator, null, Collections.emptyList()); + } + + public DataCache( + TypeSerializer<T> serializer, + FileSystem fileSystem, + SupplierWithException<Path, IOException> pathGenerator, + MemorySegmentPool segmentPool) + throws IOException { + this(serializer, fileSystem, pathGenerator, segmentPool, Collections.emptyList()); + } + + public DataCache( + TypeSerializer<T> serializer, + FileSystem fileSystem, + SupplierWithException<Path, IOException> pathGenerator, + MemorySegmentPool segmentPool, + List<Segment> finishedSegments) + throws IOException { + this.fileSystem = fileSystem; + this.pathGenerator = pathGenerator; + this.segmentPool = segmentPool; + this.serializer = serializer; + this.finishedSegments = new ArrayList<>(); + this.finishedSegments.addAll(finishedSegments); + for (Segment segment : finishedSegments) { + tryCacheSegmentToMemory(segment); + } + this.currentWriter = createSegmentWriter(); + } + + public void addRecord(T record) throws IOException { + try { + currentWriter.addRecord(record); + } catch (SegmentNoVacancyException e) { + currentWriter.finish().ifPresent(finishedSegments::add); + currentWriter = new FileSegmentWriter<>(serializer, pathGenerator.get()); + currentWriter.addRecord(record); + } + } + + /** Finishes adding records and closes resources occupied for adding records. */ + public void finish() throws IOException { + if (currentWriter == null) { + return; + } + + currentWriter.finish().ifPresent(finishedSegments::add); + currentWriter = null; + } + + /** Cleans up all previously added records. */ + public void cleanup() throws IOException { + finishCurrentSegmentIfAny(); + for (Segment segment : finishedSegments) { + if (segment.getFileSegment() != null) { + fileSystem.delete(segment.getFileSegment().getPath(), false); + } + if (segment.getMemorySegment() != null) { + segmentPool.returnAll(segment.getMemorySegment().getCache()); + } + } + finishedSegments.clear(); + } + + private void finishCurrentSegmentIfAny() throws IOException { Review Comment: There is no such usage for now. I agree that we could put `finish()` an internal method and call it before reading. -- 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. To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org