Skip to content
New issue

Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.

By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.

Already on GitHub? Sign in to your account

[KAFKA-8522] Streamline tombstone and transaction marker removal #7884

Closed
wants to merge 55 commits into from
Closed
Changes from 1 commit
Commits
Show all changes
55 commits
Select commit Hold shift + click to select a range
fdf7095
[KAFKA-8522] Streamline tombstone and transaction marker removal
ConcurrencyPractitioner Jan 2, 2020
f7dcaac
Fixing stuff
ConcurrencyPractitioner Jan 2, 2020
2ab16af
Fixing stuff
ConcurrencyPractitioner Jan 3, 2020
d36f776
Resolving some comments
ConcurrencyPractitioner Jan 15, 2020
3b2193b
Resolving remaining comments
ConcurrencyPractitioner Jan 15, 2020
dcc2f65
Adding two pass modification
ConcurrencyPractitioner Jan 26, 2020
a9d8c4d
Adding some last changes
ConcurrencyPractitioner Jan 29, 2020
dd9ca28
Adding stuff
ConcurrencyPractitioner Feb 12, 2020
1587462
Getting modified
ConcurrencyPractitioner Feb 12, 2020
a25187b
Delete total.out
ConcurrencyPractitioner Feb 12, 2020
f469515
Delete .gitignore
ConcurrencyPractitioner Feb 12, 2020
15d9d8a
Delete .gitignore
ConcurrencyPractitioner Feb 12, 2020
a3bc996
Delete LogCleaner.scala.orig
ConcurrencyPractitioner Feb 12, 2020
e00f37c
Delete diff.out
ConcurrencyPractitioner Feb 12, 2020
331ebad
Delete LogCleanerManager.scala.orig
ConcurrencyPractitioner Feb 12, 2020
4b12ebb
Delete .gitignore
ConcurrencyPractitioner Feb 12, 2020
2fc90d3
Delete .gitignore
ConcurrencyPractitioner Feb 12, 2020
041e867
Delete .gitignore
ConcurrencyPractitioner Feb 12, 2020
9df0e70
Delete .gitignore
ConcurrencyPractitioner Feb 12, 2020
613d17d
Delete .gitignore
ConcurrencyPractitioner Feb 12, 2020
69b5240
Delete .gitignore
ConcurrencyPractitioner Feb 12, 2020
ad4ff10
Delete .gitignore
ConcurrencyPractitioner Feb 12, 2020
8c9b50d
Delete .gitignore
ConcurrencyPractitioner Feb 12, 2020
de5d0a1
Delete .gitignore
ConcurrencyPractitioner Feb 12, 2020
5b43e43
Delete .gitignore
ConcurrencyPractitioner Feb 12, 2020
f665275
Delete .gitignore
ConcurrencyPractitioner Feb 12, 2020
e570f6c
Delete .gitignore
ConcurrencyPractitioner Feb 12, 2020
3c96d55
Delete .gitignore
ConcurrencyPractitioner Feb 12, 2020
a78c563
Delete org.apache.kafka.connect.rest.ConnectRestExtension
ConcurrencyPractitioner Feb 12, 2020
e287b49
Getting modified
ConcurrencyPractitioner Feb 12, 2020
ee67247
Last renaming
ConcurrencyPractitioner Feb 12, 2020
5bedf9c
Addressing some other comments
ConcurrencyPractitioner Feb 19, 2020
bd3e18f
Removing dead code
ConcurrencyPractitioner Feb 19, 2020
4515e7d
Resolving last comments
ConcurrencyPractitioner Feb 19, 2020
60d72d0
One liner
ConcurrencyPractitioner Feb 19, 2020
92530fe
Fixing broken test
ConcurrencyPractitioner Feb 19, 2020
8baa416
Adding changed files
ConcurrencyPractitioner Feb 21, 2020
6d011ed
Cleaning up messy code
ConcurrencyPractitioner Feb 26, 2020
6cc19fe
Adding some test modifications
ConcurrencyPractitioner Feb 27, 2020
8e6f9a2
Adding some further test modifications
ConcurrencyPractitioner Feb 28, 2020
d6dd028
Making some changes to test
ConcurrencyPractitioner Mar 1, 2020
9335c36
Making one liner
ConcurrencyPractitioner Mar 1, 2020
3541ea2
Adding last comments
ConcurrencyPractitioner Mar 7, 2020
a88dc20
Bumping checkstyle
ConcurrencyPractitioner Mar 7, 2020
bc3f867
Adding some spotbug fixes
ConcurrencyPractitioner Mar 8, 2020
6a1b3da
Adding some temporary log statements
ConcurrencyPractitioner Mar 9, 2020
d7491c1
Rolling back changes
ConcurrencyPractitioner Mar 10, 2020
06d9ff5
Addressing most comments
ConcurrencyPractitioner Mar 17, 2020
bed40ab
Addressing last comments
ConcurrencyPractitioner Mar 19, 2020
54cb56b
Create .asf.yml
ConcurrencyPractitioner Apr 20, 2020
81ba24c
Merge remote-tracking branch 'upstream/trunk' into KAFKA-8522
ConcurrencyPractitioner Apr 20, 2020
cadee72
Adding github whitelist
ConcurrencyPractitioner Apr 20, 2020
90877fc
Merge branch 'KAFKA-8522' of https://github.com/ConcurrencyPractition…
ConcurrencyPractitioner Apr 20, 2020
e694f13
Removing .yml file
ConcurrencyPractitioner Apr 20, 2020
a9316ad
Reverting unnecessary change
ConcurrencyPractitioner Apr 20, 2020
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
Prev Previous commit
Next Next commit
Getting modified
ConcurrencyPractitioner committed Feb 12, 2020

Verified

This commit was signed with the committer’s verified signature.
ImgBotApp Imgbot
commit 1587462f676afca5d35e6d5fe007c15ec515c3af
Original file line number Diff line number Diff line change
@@ -195,11 +195,12 @@ private static FilterResult filterTo(TopicPartition partition, Iterable<MutableR
// in which case, we need to reset the base timestamp and overwrite the timestamp deltas
// if the batch does not contain tombstones, then we don't need to overwrite batch
boolean canControlBatchBeRemoved = batch.isControlBatch() && deleteHorizonMs > RecordBatch.NO_TIMESTAMP;
if (writeOriginalBatch && (batch.deleteHorizonSet() || (!containsTombstonesOrMarker && !canControlBatchBeRemoved))) {
if (writeOriginalBatch && (deleteHorizonMs == RecordBatch.NO_TIMESTAMP || deleteHorizonMs == batch.deleteHorizonMs()
|| (!containsTombstonesOrMarker && !canControlBatchBeRemoved))) {
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

It seems that the logic can be simplified a bit. It seems that we can do this branch if writeOriginalBatch is true and needToSetDeleteHorizon is false (needToSetDeleteHorizon = (batch magic >= V2 && containsTombstonesOrMarker && batch's deleteHorizon not set)).

Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Oh, sure, that's fine. But we also still need to account for the control batch and check whether or not it is empty yet.

Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

For a control batch, it's only removed at the batch level. So, if the batch can be deleted at the batch level, we won't get in here. If the batch can't be deleted at the batch level, the record within the batch will always be retained.

Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

@junrao Is this always the case? If I remember correctly in the KIP, control batches, if it contains only tombstones, will be persisted in the logs for a set period of time i.e. we need to at some point remove the tombstones first before the control batches can be deleted. Therefore, I think it would be very much possible that we need to check for isControlBatchEmpty here.

Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

@ConcurrencyPractitioner : A control batch has only a single marker record (either a commit or abort). When all records before the control batch are removed, we set the deleteHorizon for the control batch. When the time passes the deleteHorizon, the control batch is removed. A control batch never contains a tombstone.

batch.writeTo(bufferOutputStream);
filterResult.updateRetainedBatchMetadata(batch, retainedRecords.size(), false);
} else {
final MemoryRecordsBuilder builder = buildRetainedRecordsInto(batch, retainedRecords, bufferOutputStream, deleteHorizonMs);
MemoryRecordsBuilder builder = buildRetainedRecordsInto(batch, retainedRecords, bufferOutputStream, deleteHorizonMs);
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Hmm, it seems that we only want to pass in deleteHorizonMs if containsTombstonesOrMarker && deleteHorizon is not set.

Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

By current logic, this would actually break the code. Since we don't pass a deleteHorizonSet boolean flag into the MemoryRecordsBuilder constructor, the MemoryRecordsBuilder class's current logic actually relies on the passed in argument to tell if the delete horizon has been set or not. i.e. (if deleteHorizonMs > 0L, then we set delete horizon, else we assume that it has not been set). Should I change the code correspondingly to accomadate your comment? @junrao

Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Yes, it's just that in this PR, retrieveDeleteHorizon() returns deleteHorizonMs > 0 even for batches where deleteHorizonMs doesn't need to be set. Then, we will be setting deleteHorizonMs for those batches unnecessarily.

MemoryRecords records = builder.build();
int filteredBatchSize = records.sizeInBytes();
if (filteredBatchSize > batch.sizeInBytes() && filteredBatchSize > maxRecordBatchSize)
@@ -241,7 +242,7 @@ private static BatchIterationResult iterateOverBatch(RecordBatch batch,
FilterResult filterResult,
RecordFilter filter,
byte batchMagic,
boolean recordsFiltered,
boolean writeOriginalBatch,
long maxOffset,
List<Record> retainedRecords) {
boolean containsTombstonesOrMarker = false;
@@ -254,7 +255,7 @@ private static BatchIterationResult iterateOverBatch(RecordBatch batch,
// Check for log corruption due to KAFKA-4298. If we find it, make sure that we overwrite
// the corrupted batch with correct data.
if (!record.hasMagic(batchMagic))
recordsFiltered = false;
writeOriginalBatch = false;

if (record.offset() > maxOffset)
maxOffset = record.offset();
@@ -265,10 +266,10 @@ private static BatchIterationResult iterateOverBatch(RecordBatch batch,
containsTombstonesOrMarker = true;
}
} else {
recordsFiltered = false;
writeOriginalBatch = false;
}
}
return new BatchIterationResult(recordsFiltered, containsTombstonesOrMarker, maxOffset);
return new BatchIterationResult(writeOriginalBatch, containsTombstonesOrMarker, maxOffset);
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

It's probably better to name writeOriginalBatch here to sth like recordsFiltered since we combine other information to determine writeOriginalBatch later on.

}
}

Original file line number Diff line number Diff line change
@@ -208,9 +208,7 @@ class LogCleanerIntegrationTest extends AbstractLogCleanerIntegrationTest with K

// We sleep a little bit, so that log cleaner has already gone through
// some iterations, ensures that delete horizons has been updated correctly
Thread.sleep(400L)
assertEquals(log.latestDeleteHorizon, T0 + tombstoneRetentionMs)

Thread.sleep(300L)
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

This seems unnecessary since we are waiting in cleaner.awaitCleaned() already later.

Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Yeah, it definitely is inconsistent with other tests in that there is a Thread.sleep(). Problem is that this test seems prone to be somewhat flaky. Without the sleep, at the present state, it definitely fails.

time.sleep(tombstoneRetentionMs + 1)

val latestOffset: Long = log.latestEpoch match {
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

This seems to be a complicated way of getting latestOffset. We could just do log.logEndOffset.

@@ -231,10 +229,10 @@ class LogCleanerIntegrationTest extends AbstractLogCleanerIntegrationTest with K
cleaner.awaitCleaned(new TopicPartition("log-partition", 0),
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

This is unnecessary given the waitUntilTrue() below.

latestOffset + 1, maxWaitMs = tombstoneRetentionMs)
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

To avoid transient failures, we probably want to give long enough maxWaitMs, sth like 5 secs.


assertEquals(log.latestDeleteHorizon, RecordBatch.NO_TIMESTAMP)
for (segment <- log.logSegments; record <- segment.log.records.asScala) {
fail ("The log should not contain record " + record + ", tombstone has expired its lifetime.")
}
assertEquals(log.latestDeleteHorizon, -1L)
}

private def readFromLog(log: Log): Iterable[(Int, Int)] = {
1 change: 0 additions & 1 deletion gradle.properties
Original file line number Diff line number Diff line change
@@ -23,5 +23,4 @@ group=org.apache.kafka
version=2.5.0-SNAPSHOT
scalaVersion=2.12.10
task=build
org.gradle.caching=true
org.gradle.jvmargs=-Xmx1024m -Xss2m