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

Pauseless Consumption #14460

Draft
wants to merge 31 commits into
base: master
Choose a base branch
from
Draft

Pauseless Consumption #14460

wants to merge 31 commits into from

Conversation

9aman
Copy link
Contributor

@9aman 9aman commented Nov 15, 2024

[DRAFT]

The PR covers changes for to allow consumption during segment build and upload.

Following are the core changes:

Commit Protocol Changes:

Additional parameter to signify that the pauseless is enabled.
- Need: This has been added to prevent issues that might arise due to change in table config during Commit protocol.
- Alternative: Fetch the value from tableConfig instead.

Server Side Changes

RealtimeSegmentDataManager

- Changes in the sequence of steps followed during the commit protocol to allow ingestion during build and upload. 

RealtimeTableDataManager

- Waited download in case of absence of download url.

Controller Side Changes

SegmentCompletionManager

- Changes in FSM to reflect changes in the commit protocol sequence. 
- Allow replicas to also proceed with ingestion during the build and upload.

PinotLLCRealtimeSegmentManager

- Changes to update the ZK and Ideal State as per the new protocol i.e. marking segment COMMITTING.

Aman Khanchandani and others added 30 commits November 15, 2024 09:01
1. Segment commit won't create a new segment for the pauseless path as the segment has already been created at the commit start
2. Segment commit will create a new segment if the conventional path is chosen
… is a controller failure.

The server would just now persist the segment to disk instead of retrying.
@@ -485,8 +485,7 @@ private void setUpPinotController() {

// Helix resource manager must be started in order to create PinotLLCRealtimeSegmentManager
LOGGER.info("Starting realtime segment manager");
_pinotLLCRealtimeSegmentManager =
new PinotLLCRealtimeSegmentManager(_helixResourceManager, _config, _controllerMetrics);
_pinotLLCRealtimeSegmentManager = createPinotLLCRealtimeSegmentManager();
Copy link
Contributor Author

Choose a reason for hiding this comment

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

These changes in the ControllerStarter have been added to easily override two crucial classes:
PinotLLCRealtimeSegmentManager and RealtimeSegmentValidationManager.

// TODO (akkhanch): introducing this as the first step of the segment metadata might have succeeded. We don't
// want the server to try indefinitely, rather we would rely on the validation manager to complete the remaining
// steps.
if (segmentMetadata.getStatus() == CommonConstants.Segment.Realtime.Status.DONE
Copy link
Contributor Author

Choose a reason for hiding this comment

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

This has been added to prevent servers from retrying indefinitely.

@@ -586,8 +621,21 @@ public SegmentCompletionProtocol.Response extendBuildTime(final String instanceI
case COMMITTER_DECIDED:
return fail(instanceId, offset);
case COMMITTER_NOTIFIED:
return committerNotifiedExtendBuildTime(instanceId, offset, extTimeSec, now);
if (!_pauselessConsumptionEnabled) {
return committerNotifiedExtendBuildTime(instanceId, offset, extTimeSec, now);
Copy link
Contributor Author

Choose a reason for hiding this comment

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

Change in the commit protocol changes the state in which we get the extendBuildTime request

@@ -1015,7 +1076,12 @@ private SegmentCompletionProtocol.Response processConsumedAfterCommitStart(Strin
// Common case: A different instance is reporting.
if (offset.compareTo(_winningOffset) == 0) {
// Wait until winner has posted the segment before asking this server to KEEP the segment.
response = hold(instanceId, offset);
// Keep if it's pauseless enabled
Copy link
Contributor Author

Choose a reason for hiding this comment

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

We allow the replicas to build the segment. This ensures that:

  1. Reduced need for disaster recovery: The server is persisted to disk on atleast one of the servers.
  2. Continued ingestion on replicas: The replicas can continue ingesting newly created segments rather than waiting for the committing server to complete the commit (existing behavior).


// TODO: might need to support the two ways in which stream config can be set. On under the ingestion
// config and other under the tableIndexConfig
if (isPauselessEnabeld()) {
Copy link
Contributor Author

Choose a reason for hiding this comment

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

We want to Initiate the COMMIT_START before building the segment. This is crucial for allowing ingestion while the current segment is being built.

@codecov-commenter
Copy link

codecov-commenter commented Nov 15, 2024

Codecov Report

Attention: Patch coverage is 18.88889% with 73 lines in your changes missing coverage. Please review.

Project coverage is 55.57%. Comparing base (59551e4) to head (9dfcc4e).
Report is 1335 commits behind head on master.

Files with missing lines Patch % Lines
...ata/manager/realtime/RealtimeTableDataManager.java 0.00% 32 Missing ⚠️
...a/manager/realtime/RealtimeSegmentDataManager.java 3.22% 27 Missing and 3 partials ⚠️
...e/data/manager/realtime/SplitSegmentCommitter.java 0.00% 6 Missing ⚠️
...ot/common/protocols/SegmentCompletionProtocol.java 66.66% 2 Missing ⚠️
...he/pinot/spi/utils/builder/TableConfigBuilder.java 33.33% 2 Missing ⚠️
.../pinot/core/data/manager/BaseTableDataManager.java 66.66% 0 Missing and 1 partial ⚠️

❗ There is a different number of reports uploaded between BASE (59551e4) and HEAD (9dfcc4e). Click for more details.

HEAD has 7 uploads less than BASE
Flag BASE (59551e4) HEAD (9dfcc4e)
skip-bytebuffers-false 7 6
unittests 5 3
java-11 5 4
unittests2 3 0
Additional details and impacted files
@@             Coverage Diff              @@
##             master   #14460      +/-   ##
============================================
- Coverage     61.75%   55.57%   -6.18%     
- Complexity      207      796     +589     
============================================
  Files          2436     2099     -337     
  Lines        133233   110518   -22715     
  Branches      20636    17528    -3108     
============================================
- Hits          82274    61420   -20854     
+ Misses        44911    44234     -677     
+ Partials       6048     4864    -1184     
Flag Coverage Δ
custom-integration1 100.00% <ø> (+99.99%) ⬆️
integration 100.00% <ø> (+99.99%) ⬆️
integration1 100.00% <ø> (+99.99%) ⬆️
integration2 0.00% <ø> (ø)
java-11 55.53% <18.88%> (-6.18%) ⬇️
java-21 55.43% <18.88%> (-6.19%) ⬇️
skip-bytebuffers-false 55.56% <18.88%> (-6.19%) ⬇️
skip-bytebuffers-true 55.41% <18.88%> (+27.68%) ⬆️
temurin 55.57% <18.88%> (-6.18%) ⬇️
unittests 55.57% <18.88%> (-6.18%) ⬇️
unittests1 55.57% <18.88%> (+8.68%) ⬆️
unittests2 ?

Flags with carried forward coverage won't be shown. Click here to find out more.

☔ View full report in Codecov by Sentry.
📢 Have feedback on the report? Share it here.

Sign up for free to join this conversation on GitHub. Already have an account? Sign in to comment
Labels
None yet
Projects
None yet
Development

Successfully merging this pull request may close these issues.

2 participants