package org.apache.lucene.util;
import java.io.Closeable;
import java.io.EOFException;
import java.io.IOException;
import java.util.ArrayList;
import java.util.Comparator;
import java.util.List;
import java.util.Locale;
import java.util.concurrent.Callable;
import java.util.concurrent.ExecutionException;
import java.util.concurrent.ExecutorService;
import java.util.concurrent.Future;
import java.util.concurrent.Semaphore;
import java.util.concurrent.atomic.AtomicLong;
import org.apache.lucene.codecs.CodecUtil;
import org.apache.lucene.store.ChecksumIndexInput;
import org.apache.lucene.store.Directory;
import org.apache.lucene.store.IOContext;
import org.apache.lucene.store.IndexInput;
import org.apache.lucene.store.IndexOutput;
import org.apache.lucene.store.TrackingDirectoryWrapper;
public class OfflineSorter {
public final static long MB = 1024 * 1024;
public final static long GB = MB * 1024;
public final static long MIN_BUFFER_SIZE_MB = 32;
public static final long ABSOLUTE_MIN_SORT_BUFFER_SIZE = MB / 2;
private static final String MIN_BUFFER_SIZE_MSG = "At least 0.5MB RAM buffer is needed";
public final static int MAX_TEMPFILES = 10;
private final Directory dir;
private final int valueLength;
private final String tempFileNamePrefix;
private final ExecutorService exec;
private final Semaphore partitionsInRAM;
public static final class BufferSize {
final int bytes;
private BufferSize(long bytes) {
if (bytes > Integer.MAX_VALUE) {
throw new IllegalArgumentException("Buffer too large for Java ("
+ (Integer.MAX_VALUE / MB) + "mb max): " + bytes);
}
if (bytes < ABSOLUTE_MIN_SORT_BUFFER_SIZE) {
throw new IllegalArgumentException(MIN_BUFFER_SIZE_MSG + ": " + bytes);
}
this.bytes = (int) bytes;
}
public static BufferSize megabytes(long mb) {
return new BufferSize(mb * MB);
}
public static BufferSize automatic() {
Runtime rt = Runtime.getRuntime();
final long max = rt.maxMemory();
final long total = rt.totalMemory();
final long free = rt.freeMemory();
final long totalAvailableBytes = max - total + free;
long sortBufferByteSize = free/2;
final long minBufferSizeBytes = MIN_BUFFER_SIZE_MB*MB;
if (sortBufferByteSize < minBufferSizeBytes
|| totalAvailableBytes > 10 * minBufferSizeBytes) {
if (totalAvailableBytes/2 > minBufferSizeBytes) {
sortBufferByteSize = totalAvailableBytes/2;
} else {
sortBufferByteSize = Math.max(ABSOLUTE_MIN_SORT_BUFFER_SIZE, sortBufferByteSize);
}
}
return new BufferSize(Math.min((long)Integer.MAX_VALUE, sortBufferByteSize));
}
}
public class SortInfo {
public int tempMergeFiles;
public int mergeRounds;
public int lineCount;
public final AtomicLong mergeTimeMS = new AtomicLong();
public final AtomicLong sortTimeMS = new AtomicLong();
public long totalTimeMS;
public long readTimeMS;
public final long bufferSize = ramBufferSize.bytes;
public SortInfo() {}
@Override
public String toString() {
return String.format(Locale.ROOT,
"time=%.2f sec. total (%.2f reading, %.2f sorting, %.2f merging), lines=%d, temp files=%d, merges=%d, soft ram limit=%.2f MB",
totalTimeMS / 1000.0d, readTimeMS / 1000.0d, sortTimeMS.get() / 1000.0d, mergeTimeMS.get() / 1000.0d,
lineCount, tempMergeFiles, mergeRounds,
(double) bufferSize / MB);
}
}
private final BufferSize ramBufferSize;
SortInfo sortInfo;
private int maxTempFiles;
private final Comparator<BytesRef> comparator;
public static final Comparator<BytesRef> DEFAULT_COMPARATOR = Comparator.naturalOrder();
public OfflineSorter(Directory dir, String tempFileNamePrefix) throws IOException {
this(dir, tempFileNamePrefix, DEFAULT_COMPARATOR, BufferSize.automatic(), MAX_TEMPFILES, -1, null, 0);
}
public OfflineSorter(Directory dir, String tempFileNamePrefix, Comparator<BytesRef> comparator) throws IOException {
this(dir, tempFileNamePrefix, comparator, BufferSize.automatic(), MAX_TEMPFILES, -1, null, 0);
}
public OfflineSorter(Directory dir, String tempFileNamePrefix, Comparator<BytesRef> comparator,
BufferSize ramBufferSize, int maxTempfiles, int valueLength, ExecutorService exec,
int maxPartitionsInRAM) {
if (exec != null) {
this.exec = exec;
if (maxPartitionsInRAM <= 0) {
throw new IllegalArgumentException("maxPartitionsInRAM must be > 0; got " + maxPartitionsInRAM);
}
} else {
this.exec = new SameThreadExecutorService();
maxPartitionsInRAM = 1;
}
this.partitionsInRAM = new Semaphore(maxPartitionsInRAM);
if (ramBufferSize.bytes < ABSOLUTE_MIN_SORT_BUFFER_SIZE) {
throw new IllegalArgumentException(MIN_BUFFER_SIZE_MSG + ": " + ramBufferSize.bytes);
}
if (maxTempfiles < 2) {
throw new IllegalArgumentException("maxTempFiles must be >= 2");
}
if (valueLength != -1 && (valueLength == 0 || valueLength > Short.MAX_VALUE)) {
throw new IllegalArgumentException("valueLength must be 1 .. " + Short.MAX_VALUE + "; got: " + valueLength);
}
this.valueLength = valueLength;
this.ramBufferSize = ramBufferSize;
this.maxTempFiles = maxTempfiles;
this.comparator = comparator;
this.dir = dir;
this.tempFileNamePrefix = tempFileNamePrefix;
}
public Directory getDirectory() {
return dir;
}
public String getTempFileNamePrefix() {
return tempFileNamePrefix;
}
public String sort(String inputFileName) throws IOException {
sortInfo = new SortInfo();
long startMS = System.currentTimeMillis();
List<Future<Partition>> segments = new ArrayList<>();
int[] levelCounts = new int[1];
TrackingDirectoryWrapper trackingDir = new TrackingDirectoryWrapper(dir);
boolean success = false;
try (ByteSequencesReader is = getReader(dir.openChecksumInput(inputFileName, IOContext.READONCE), inputFileName)) {
while (true) {
Partition part = readPartition(is);
if (part.count == 0) {
if (partitionsInRAM != null) {
partitionsInRAM.release();
}
assert part.exhausted;
break;
}
Callable<Partition> job = new SortPartitionTask(trackingDir, part);
segments.add(exec.submit(job));
sortInfo.tempMergeFiles++;
sortInfo.lineCount += part.count;
levelCounts[0]++;
int mergeLevel = 0;
while (levelCounts[mergeLevel] == maxTempFiles) {
mergePartitions(trackingDir, segments);
if (mergeLevel+2 > levelCounts.length) {
levelCounts = ArrayUtil.grow(levelCounts, mergeLevel+2);
}
levelCounts[mergeLevel+1]++;
levelCounts[mergeLevel] = 0;
mergeLevel++;
}
if (part.exhausted) {
break;
}
}
while (segments.size() > 1) {
mergePartitions(trackingDir, segments);
}
String result;
if (segments.isEmpty()) {
try (IndexOutput out = trackingDir.createTempOutput(tempFileNamePrefix, "sort", IOContext.DEFAULT)) {
CodecUtil.writeFooter(out);
result = out.getName();
}
} else {
result = getPartition(segments.get(0)).fileName;
}
assert trackingDir.getCreatedFiles().size() == 1 && trackingDir.getCreatedFiles().contains(result);
sortInfo.totalTimeMS = System.currentTimeMillis() - startMS;
CodecUtil.checkFooter(is.in);
success = true;
return result;
} catch (InterruptedException ie) {
throw new ThreadInterruptedException(ie);
} finally {
if (success == false) {
IOUtils.deleteFilesIgnoringExceptions(trackingDir, trackingDir.getCreatedFiles());
}
}
}
private void verifyChecksum(Throwable priorException, ByteSequencesReader reader) throws IOException {
try (ChecksumIndexInput in = dir.openChecksumInput(reader.name, IOContext.READONCE)) {
CodecUtil.checkFooter(in, priorException);
}
}
void mergePartitions(Directory trackingDir, List<Future<Partition>> segments) throws IOException {
long start = System.currentTimeMillis();
List<Future<Partition>> segmentsToMerge;
if (segments.size() > maxTempFiles) {
segmentsToMerge = segments.subList(segments.size() - maxTempFiles, segments.size());
} else {
segmentsToMerge = segments;
}
sortInfo.mergeRounds++;
MergePartitionsTask task = new MergePartitionsTask(trackingDir, new ArrayList<>(segmentsToMerge));
segmentsToMerge.clear();
segments.add(exec.submit(task));
sortInfo.tempMergeFiles++;
}
private static class Partition {
public final SortableBytesRefArray buffer;
public final boolean exhausted;
public final long count;
public final String fileName;
public Partition(SortableBytesRefArray buffer, boolean exhausted) {
this.buffer = buffer;
this.fileName = null;
this.count = buffer.size();
this.exhausted = exhausted;
}
public Partition(String fileName, long count) {
this.buffer = null;
this.fileName = fileName;
this.count = count;
this.exhausted = true;
}
}
Partition readPartition(ByteSequencesReader reader) throws IOException, InterruptedException {
if (partitionsInRAM != null) {
partitionsInRAM.acquire();
}
boolean success = false;
try {
long start = System.currentTimeMillis();
SortableBytesRefArray buffer;
boolean exhausted = false;
int count;
if (valueLength != -1) {
buffer = new FixedLengthBytesRefArray(valueLength);
int limit = ramBufferSize.bytes / valueLength;
for(int i=0;i<limit;i++) {
BytesRef item = null;
try {
item = reader.next();
} catch (Throwable t) {
verifyChecksum(t, reader);
}
if (item == null) {
exhausted = true;
break;
}
buffer.append(item);
}
} else {
Counter bufferBytesUsed = Counter.newCounter();
buffer = new BytesRefArray(bufferBytesUsed);
while (true) {
BytesRef item = null;
try {
item = reader.next();
} catch (Throwable t) {
verifyChecksum(t, reader);
}
if (item == null) {
exhausted = true;
break;
}
buffer.append(item);
if (bufferBytesUsed.get() > ramBufferSize.bytes) {
break;
}
}
}
sortInfo.readTimeMS += System.currentTimeMillis() - start;
success = true;
return new Partition(buffer, exhausted);
} finally {
if (success == false && partitionsInRAM != null) {
partitionsInRAM.release();
}
}
}
static class FileAndTop {
final int fd;
BytesRef current;
FileAndTop(int fd, BytesRef firstLine) {
this.fd = fd;
this.current = firstLine;
}
}
protected ByteSequencesWriter getWriter(IndexOutput out, long itemCount) throws IOException {
return new ByteSequencesWriter(out);
}
protected ByteSequencesReader getReader(ChecksumIndexInput in, String name) throws IOException {
return new ByteSequencesReader(in, name);
}
public static class ByteSequencesWriter implements Closeable {
protected final IndexOutput out;
public ByteSequencesWriter(IndexOutput out) {
this.out = out;
}
public final void write(BytesRef ref) throws IOException {
assert ref != null;
write(ref.bytes, ref.offset, ref.length);
}
public final void write(byte[] bytes) throws IOException {
write(bytes, 0, bytes.length);
}
public void write(byte[] bytes, int off, int len) throws IOException {
assert bytes != null;
assert off >= 0 && off + len <= bytes.length;
assert len >= 0;
if (len > Short.MAX_VALUE) {
throw new IllegalArgumentException("len must be <= " + Short.MAX_VALUE + "; got " + len);
}
out.writeShort((short) len);
out.writeBytes(bytes, off, len);
}
@Override
public void close() throws IOException {
out.close();
}
}
public static class ByteSequencesReader implements BytesRefIterator, Closeable {
protected final String name;
protected final ChecksumIndexInput in;
protected final long end;
private final BytesRefBuilder ref = new BytesRefBuilder();
public ByteSequencesReader(ChecksumIndexInput in, String name) {
this.in = in;
this.name = name;
end = in.length() - CodecUtil.footerLength();
}
public BytesRef next() throws IOException {
if (in.getFilePointer() >= end) {
return null;
}
short length = in.readShort();
ref.grow(length);
ref.setLength(length);
in.readBytes(ref.bytes(), 0, length);
return ref.get();
}
@Override
public void close() throws IOException {
in.close();
}
}
public Comparator<BytesRef> getComparator() {
return comparator;
}
private class SortPartitionTask implements Callable<Partition> {
private final Directory dir;
private final Partition part;
public SortPartitionTask(Directory dir, Partition part) {
this.dir = dir;
this.part = part;
}
@Override
public Partition call() throws IOException {
try (IndexOutput tempFile = dir.createTempOutput(tempFileNamePrefix, "sort", IOContext.DEFAULT);
ByteSequencesWriter out = getWriter(tempFile, part.buffer.size());) {
BytesRef spare;
long startMS = System.currentTimeMillis();
BytesRefIterator iter = part.buffer.iterator(comparator);
sortInfo.sortTimeMS.addAndGet(System.currentTimeMillis() - startMS);
int count = 0;
while ((spare = iter.next()) != null) {
out.write(spare);
count++;
}
assert count == part.count;
CodecUtil.writeFooter(out.out);
part.buffer.clear();
return new Partition(tempFile.getName(), part.count);
} finally {
if (partitionsInRAM != null) {
partitionsInRAM.release();
}
}
}
}
private Partition getPartition(Future<Partition> future) throws IOException {
try {
return future.get();
} catch (InterruptedException ie) {
throw new ThreadInterruptedException(ie);
} catch (ExecutionException ee) {
Throwable cause = ee.getCause();
throw IOUtils.rethrowAlways(cause != null ? cause : ee);
}
}
private class MergePartitionsTask implements Callable<Partition> {
private final Directory dir;
private final List<Future<Partition>> segmentsToMerge;
public MergePartitionsTask(Directory dir, List<Future<Partition>> segmentsToMerge) {
this.dir = dir;
this.segmentsToMerge = segmentsToMerge;
}
@Override
public Partition call() throws IOException {
long totalCount = 0;
for (Future<Partition> segment : segmentsToMerge) {
totalCount += getPartition(segment).count;
}
PriorityQueue<FileAndTop> queue = new PriorityQueue<FileAndTop>(segmentsToMerge.size()) {
@Override
protected boolean lessThan(FileAndTop a, FileAndTop b) {
return comparator.compare(a.current, b.current) < 0;
}
};
ByteSequencesReader[] streams = new ByteSequencesReader[segmentsToMerge.size()];
String newSegmentName = null;
long startMS = System.currentTimeMillis();
try (ByteSequencesWriter writer = getWriter(dir.createTempOutput(tempFileNamePrefix, "sort", IOContext.DEFAULT), totalCount)) {
newSegmentName = writer.out.getName();
for (int i = 0; i < segmentsToMerge.size(); i++) {
Partition segment = getPartition(segmentsToMerge.get(i));
streams[i] = getReader(dir.openChecksumInput(segment.fileName, IOContext.READONCE), segment.fileName);
BytesRef item = null;
try {
item = streams[i].next();
} catch (Throwable t) {
verifyChecksum(t, streams[i]);
}
assert item != null;
queue.insertWithOverflow(new FileAndTop(i, item));
}
FileAndTop top;
while ((top = queue.top()) != null) {
writer.write(top.current);
try {
top.current = streams[top.fd].next();
} catch (Throwable t) {
verifyChecksum(t, streams[top.fd]);
}
if (top.current != null) {
queue.updateTop();
} else {
queue.pop();
}
}
CodecUtil.writeFooter(writer.out);
for(ByteSequencesReader reader : streams) {
CodecUtil.checkFooter(reader.in);
}
sortInfo.mergeTimeMS.addAndGet(System.currentTimeMillis() - startMS);
} finally {
IOUtils.close(streams);
}
List<String> toDelete = new ArrayList<>();
for (Future<Partition> segment : segmentsToMerge) {
toDelete.add(getPartition(segment).fileName);
}
IOUtils.deleteFiles(dir, toDelete);
return new Partition(newSegmentName, totalCount);
}
}
}