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
raftstore: Remove future_poller pool and batch Ticks #8457
Conversation
Signed-off-by: Little-Wallace <bupt2013211450@gmail.com>
@@ -691,6 +707,37 @@ impl<EK: KvEngine, ER: RaftEngine, T: Transport, C: PdClient> RaftPoller<EK, ER, | |||
self.poll_ctx.raft_metrics.ready.snapshot - self.previous_metrics.ready.snapshot | |||
); | |||
} | |||
|
|||
fn flush_ticks(&mut self) { | |||
const TICKS: &[PeerTicks] = &[ |
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.
Better use static and add a test to check if all types of PeerTicks has been included.
}); | ||
let idx = tick.bits() as usize; | ||
let batch = &mut self.ctx.tick_batch[idx]; | ||
if batch.ticks.is_empty() || batch.wait_duration == timeout { |
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.
Maybe we can batch all of the timer and combine the same timeout timer.
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.
But the timeout of most ticks are different...
@@ -691,6 +707,37 @@ impl<EK: KvEngine, ER: RaftEngine, T: Transport, C: PdClient> RaftPoller<EK, ER, | |||
self.poll_ctx.raft_metrics.ready.snapshot - self.previous_metrics.ready.snapshot | |||
); | |||
} | |||
|
|||
fn flush_ticks(&mut self) { | |||
const TICKS: &[PeerTicks] = &[ |
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.
How about adding a method to PeerTicks to return all ticks?
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.
Ok.
#[derive(Default)] | ||
pub struct PeerTickBatch { | ||
pub ticks: Vec<Box<dyn FnOnce() + Send>>, | ||
pub wait_duration: Duration, |
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.
Maybe we can remove it, for the same tick the timeout should be the same, we can get the timeout from config when flushing ticks.
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.
Ok
Signed-off-by: Little-Wallace <bupt2013211450@gmail.com>
Signed-off-by: Little-Wallace <bupt2013211450@gmail.com>
Signed-off-by: Little-Wallace <bupt2013211450@gmail.com>
/test |
@@ -374,6 +390,21 @@ where | |||
} | |||
timeout | |||
} | |||
|
|||
pub fn update_ticks_timeout(&mut self) { |
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.
Should call this function after config updated at
tikv/components/raftstore/src/store/fsm/store.rs
Lines 727 to 747 in 7ce49e7
// update config | |
self.poll_ctx.perf_context_statistics.start(); | |
if let Some(incoming) = self.cfg_tracker.any_new() { | |
match Ord::cmp( | |
&incoming.messages_per_tick, | |
&self.poll_ctx.cfg.messages_per_tick, | |
) { | |
CmpOrdering::Greater => { | |
self.store_msg_buf.reserve(incoming.messages_per_tick); | |
self.peer_msg_buf.reserve(incoming.messages_per_tick); | |
self.messages_per_tick = incoming.messages_per_tick; | |
} | |
CmpOrdering::Less => { | |
self.store_msg_buf.shrink_to(incoming.messages_per_tick); | |
self.peer_msg_buf.shrink_to(incoming.messages_per_tick); | |
self.messages_per_tick = incoming.messages_per_tick; | |
} | |
_ => {} | |
} | |
self.poll_ctx.cfg = incoming.clone(); | |
} |
or instead of maintaining wait_duration
, we can get the up-to-date config from self.cfg
through a function like fn get_wait_duration(&self, tick: PeerTicks) -> Duration
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.
Ok.
Codecov Report
@@ Coverage Diff @@
## master #8457 +/- ##
==========================================
+ Coverage 84.45% 84.55% +0.09%
==========================================
Files 596 590 -6
Lines 145303 143853 -1450
==========================================
- Hits 122714 121629 -1085
+ Misses 22589 22224 -365
Continue to review full report at Codecov.
|
Signed-off-by: Little-Wallace <bupt2013211450@gmail.com>
Signed-off-by: Little-Wallace <bupt2013211450@gmail.com>
Signed-off-by: Little-Wallace <bupt2013211450@gmail.com>
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.
LGTM
@NingLin-P,Thanks for your review. The bot only counts LGTMs from Reviewers and higher roles, but you're still welcome to leave your comments.See the corresponding SIG page for more information. Related SIG: raft(slack). |
/test |
1 similar comment
/test |
Signed-off-by: Little-Wallace <bupt2013211450@gmail.com>
Signed-off-by: Little-Wallace <bupt2013211450@gmail.com>
@@ -158,8 +158,6 @@ fn test_tick_after_destroy() { | |||
let engine_3 = cluster.get_engine(3); | |||
must_get_equal(&engine_3, b"k1", b"v1"); | |||
|
|||
let tick_fp = "on_raft_log_gc_tick_1"; | |||
fail::cfg(tick_fp, "pause").unwrap(); |
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.
Change this failpoint to function on_raft_gc_log_tick
and use return
not pause
then set raft_log_gc_tick_interval
to 50ms in this test.
Signed-off-by: Little-Wallace <bupt2013211450@gmail.com>
@@ -176,9 +176,9 @@ fn test_tick_after_destroy() { | |||
cluster.must_put(b"k3", b"v3"); | |||
|
|||
thread::sleep(cluster.cfg.raft_store.raft_log_gc_tick_interval.0); | |||
fail::remove(tick_fp); |
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.
The order should not be changed.
d3d2b1e
to
2760123
Compare
Signed-off-by: Little-Wallace <bupt2013211450@gmail.com>
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.
LGTM
@Little-Wallace Oops! auto merge is restricted to Committers of the SIG.See the corresponding SIG page for more information. Related SIG: raft(slack). |
/merge |
@gengliqi Oops! This PR requires at least 2 LGTMs to merge. The current number of |
Signed-off-by: ti-srebot <ti-srebot@pingcap.com>
cherry pick to release-4.0 in PR #8634 |
Signed-off-by: Little-Wallace <bupt2013211450@gmail.com>
Signed-off-by: Little-Wallace bupt2013211450@gmail.com
What problem does this PR solve?
To reduce the cost of TiKV heartbeat tick.
What is changed and how it works?
Batch all ticks in once
poll
together and call them in delay thread rather than future-poller.Related changes
pingcap/docs
/pingcap/docs-cn
:pingcap/tidb-ansible
:Check List
Tests
Side effects
Release note