Skip to content

Commit

Permalink
changes for exprimenting w/ drop behind
Browse files Browse the repository at this point in the history
  • Loading branch information
keith-turner committed Nov 25, 2022
1 parent 67012a5 commit 4a0795f
Show file tree
Hide file tree
Showing 5 changed files with 22 additions and 14 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -969,6 +969,11 @@ public enum Property {
"Setting this property to true will call"
+ "FSDataOutputStream.setDropBehind(true) on the major compaction output stream.",
"2.1.1"),
TABLE_MAJC_INPUT_DROP_CACHE("table.compaction.major.input.drop.cache", "false",
PropertyType.BOOLEAN,
"Setting this property to true will call"
+ "FSDataOutputStream.setDropBehind(true) on the major compaction input stream(s).",
"2.1.1"),
TABLE_MAJC_RATIO("table.compaction.major.ratio", "3", PropertyType.FRACTION,
"Minimum ratio of total input size to maximum input RFile size for"
+ " running a major compaction. ",
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -390,8 +390,8 @@ public WriterBuilder withRateLimiter(RateLimiter rateLimiter) {
return this;
}

public WriterBuilder dropCachesBehind() {
this.dropCacheBehind(true);
public WriterBuilder dropCachesBehind(boolean shouldDrop) {
this.dropCacheBehind(shouldDrop);
return this;
}

Expand Down Expand Up @@ -443,8 +443,8 @@ public ReaderBuilder withRateLimiter(RateLimiter rateLimiter) {
return this;
}

public ReaderBuilder dropCachesBehind() {
this.dropCacheBehind(true);
public ReaderBuilder dropCachesBehind(boolean shouldDrop) {
this.dropCacheBehind(shouldDrop);
return this;
}

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -96,7 +96,7 @@ public CachableBuilder fsPath(FileSystem fs, Path dataFile, boolean dropCacheBeh
// cache
try {
is.setDropBehind(Boolean.TRUE);
log.trace("Called setDropBehind(TRUE) for stream reading file {}", dataFile);
log.debug("Called setDropBehind(TRUE) for stream reading file {}", dataFile);
} catch (UnsupportedOperationException e) {
log.debug("setDropBehind not enabled for wal file: {}", dataFile);
} catch (IOException e) {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -140,7 +140,7 @@ protected FileSKVWriter openWriter(FileOptions options) throws IOException {
// cache
try {
outputStream.setDropBehind(Boolean.TRUE);
LOG.trace("Called setDropBehind(TRUE) for stream writing file {}", options.filename);
LOG.debug("Called setDropBehind(TRUE) for stream writing file {}", options.filename);
} catch (UnsupportedOperationException e) {
LOG.debug("setDropBehind not enabled for file: {}", options.filename);
} catch (IOException e) {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -42,7 +42,6 @@
import org.apache.accumulo.core.data.Value;
import org.apache.accumulo.core.dataImpl.KeyExtent;
import org.apache.accumulo.core.file.FileOperations;
import org.apache.accumulo.core.file.FileOperations.WriterBuilder;
import org.apache.accumulo.core.file.FileSKVIterator;
import org.apache.accumulo.core.file.FileSKVWriter;
import org.apache.accumulo.core.iterators.IteratorUtil.IteratorScope;
Expand Down Expand Up @@ -224,13 +223,10 @@ public CompactionStats call() throws IOException, CompactionCanceledException {
dropCacheBehindMajcOutput = true;
}

WriterBuilder outBuilder = fileFactory.newWriterBuilder()
mfw = fileFactory.newWriterBuilder()
.forFile(outputFile.getMetaInsert(), ns, ns.getConf(), cryptoService)
.withTableConfiguration(acuTableConf).withRateLimiter(env.getWriteLimiter());
if (dropCacheBehindMajcOutput) {
outBuilder.dropCachesBehind();
}
mfw = outBuilder.build();
.withTableConfiguration(acuTableConf).withRateLimiter(env.getWriteLimiter())
.dropCachesBehind(dropCacheBehindMajcOutput).build();

Map<String,Set<ByteSequence>> lGroups = getLocalityGroups(acuTableConf);

Expand Down Expand Up @@ -320,6 +316,13 @@ public CompactionStats call() throws IOException, CompactionCanceledException {

List<SortedKeyValueIterator<Key,Value>> iters = new ArrayList<>(filesToCompact.size());

boolean dropCacheBehindMajcInput = false;
if (!RootTable.ID.equals(this.extent.tableId())
&& !MetadataTable.ID.equals(this.extent.tableId())
&& acuTableConf.getBoolean(Property.TABLE_MAJC_INPUT_DROP_CACHE)) {
dropCacheBehindMajcInput = true;
}

for (TabletFile mapFile : filesToCompact.keySet()) {
try {

Expand All @@ -330,7 +333,7 @@ public CompactionStats call() throws IOException, CompactionCanceledException {
reader = fileFactory.newReaderBuilder()
.forFile(mapFile.getPathStr(), fs, fs.getConf(), cryptoService)
.withTableConfiguration(acuTableConf).withRateLimiter(env.getReadLimiter())
.dropCachesBehind().build();
.dropCachesBehind(dropCacheBehindMajcInput).build();

readers.add(reader);

Expand Down

0 comments on commit 4a0795f

Please sign in to comment.