-
Notifications
You must be signed in to change notification settings - Fork 2.5k
[HUDI-5504]Fix concurrency conflict when asyncCompaction is enabled #7609
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
Conversation
| if (asyncCompaction) { | ||
| synchronized (mutex) { | ||
| super.processLatencyMarker(latencyMarker); | ||
| } |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
Can we just ignore the watermark and latency marker totally ? The compaction operators belongs to the sink ppeline, there is no need to propagate them to downstream.
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
In my opinion, it will be more user-friendly to display through flink ui after delivery, and some specific functions may need to deliver this information. So i do not ignore the watermark and latency marker totally.
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
now, i have ignored the watermark and latency marker totally. @danny0405
|
@hudi-bot re-run the last Azure build |
| package org.apache.hudi.sink.compact; | ||
|
|
||
| import org.apache.flink.streaming.runtime.streamrecord.LatencyMarker; | ||
| import org.apache.hudi.client.HoodieFlinkWriteClient; |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
Fix the import sequence.
| @Override | ||
| public void processWatermark(Watermark mark) { | ||
| public void processWatermark(Watermark mark) throws Exception { | ||
| // no need to propagate the watermark |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
Unnecessary change.
| @Override | ||
| public void processLatencyMarker(LatencyMarker latencyMarker) | ||
| throws Exception { | ||
| // no need to propagate the latencyMarker |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
No need to throw exception.
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
fixed
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
ClusteringOperator should also be fixed.
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
ClusteringOperator fixed in HUDI-5515
| () -> doCompaction(instantTime, compactionOperation, collector, reloadWriteConfig()), | ||
| (errMsg, t) -> collector.collect(new CompactionCommitEvent(instantTime, compactionOperation.getFileId(), taskID)), | ||
| (errMsg, t) -> { | ||
| collector.collect(new CompactionCommitEvent(instantTime, |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
Unnecessary change.
| instantTime, maxInstantTime, | ||
| writeClient.getHoodieTable().getTaskContextSupplier()); | ||
| collector.collect(new CompactionCommitEvent(instantTime, compactionOperation.getFileId(), writeStatuses, taskID)); | ||
| collector.collect(new CompactionCommitEvent(instantTime, compactionOperation.getFileId(), |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
Unnecessary change.
|
@hudi-bot run azure |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
+1, nice catch ~ We can land this once ClusteringOperator is also fixed.
…latency marker (apache#7609) Co-authored-by: coder_wang <[email protected]>
…latency marker (apache#7609) Co-authored-by: coder_wang <[email protected]>
…latency marker (apache#7609) Co-authored-by: coder_wang <[email protected]>
Change Logs
Fix concurrency conflict when asyncCompaction is enabled
Impact
When asyncCompaction is enabled , CompactTaskMainThread and compactionThread will cause concurrency conflict when send watermark and compactionCommitEvent
Risk level (write none, low medium or high below)
high
Documentation Update
Contributor's checklist