2021-10-14 09:50:12 +00:00
|
|
|
//! Contain structs related to making RPCs
|
|
|
|
use std::sync::Arc;
|
2022-02-22 12:53:59 +00:00
|
|
|
use std::time::{Duration};
|
2021-10-14 09:50:12 +00:00
|
|
|
|
|
|
|
use futures::future::join_all;
|
|
|
|
use futures::stream::futures_unordered::FuturesUnordered;
|
|
|
|
use futures::stream::StreamExt;
|
|
|
|
use futures_util::future::FutureExt;
|
|
|
|
use tokio::select;
|
2021-11-04 15:04:26 +00:00
|
|
|
use tokio::sync::{watch, Semaphore};
|
2021-10-14 09:50:12 +00:00
|
|
|
|
2022-02-17 22:28:23 +00:00
|
|
|
use opentelemetry::KeyValue;
|
|
|
|
use opentelemetry::{
|
|
|
|
trace::{FutureExt as OtelFutureExt, Span, TraceContextExt, Tracer},
|
|
|
|
Context,
|
|
|
|
};
|
|
|
|
|
2021-10-15 09:05:09 +00:00
|
|
|
pub use netapp::endpoint::{Endpoint, EndpointHandler, Message as Rpc};
|
2021-10-14 09:50:12 +00:00
|
|
|
use netapp::peering::fullmesh::FullMeshPeeringStrategy;
|
|
|
|
pub use netapp::proto::*;
|
|
|
|
pub use netapp::{NetApp, NodeID};
|
|
|
|
|
|
|
|
use garage_util::background::BackgroundRunner;
|
2021-11-03 16:00:40 +00:00
|
|
|
use garage_util::data::*;
|
2021-10-19 14:16:10 +00:00
|
|
|
use garage_util::error::Error;
|
2022-02-22 12:53:59 +00:00
|
|
|
use garage_util::metrics::RecordDuration;
|
2021-10-14 09:50:12 +00:00
|
|
|
|
2022-02-16 13:23:04 +00:00
|
|
|
use crate::metrics::RpcMetrics;
|
2021-11-04 15:04:26 +00:00
|
|
|
use crate::ring::Ring;
|
|
|
|
|
2021-10-14 09:50:12 +00:00
|
|
|
const DEFAULT_TIMEOUT: Duration = Duration::from_secs(10);
|
|
|
|
|
2021-11-03 16:00:40 +00:00
|
|
|
// Try to never have more than 200MB of outgoing requests
|
|
|
|
// buffered at the same time. Other requests are queued until
|
|
|
|
// space is freed.
|
|
|
|
const REQUEST_BUFFER_SIZE: usize = 200 * 1024 * 1024;
|
|
|
|
|
2021-10-14 09:50:12 +00:00
|
|
|
/// Strategy to apply when making RPC
|
|
|
|
#[derive(Copy, Clone)]
|
|
|
|
pub struct RequestStrategy {
|
|
|
|
/// Max time to wait for reponse
|
|
|
|
pub rs_timeout: Duration,
|
|
|
|
/// Min number of response to consider the request successful
|
|
|
|
pub rs_quorum: Option<usize>,
|
|
|
|
/// Should requests be dropped after enough response are received
|
|
|
|
pub rs_interrupt_after_quorum: bool,
|
|
|
|
/// Request priority
|
|
|
|
pub rs_priority: RequestPriority,
|
|
|
|
}
|
|
|
|
|
|
|
|
impl RequestStrategy {
|
|
|
|
/// Create a RequestStrategy with default timeout and not interrupting when quorum reached
|
|
|
|
pub fn with_priority(prio: RequestPriority) -> Self {
|
|
|
|
RequestStrategy {
|
|
|
|
rs_timeout: DEFAULT_TIMEOUT,
|
|
|
|
rs_quorum: None,
|
|
|
|
rs_interrupt_after_quorum: false,
|
|
|
|
rs_priority: prio,
|
|
|
|
}
|
|
|
|
}
|
|
|
|
/// Set quorum to be reached for request
|
|
|
|
pub fn with_quorum(mut self, quorum: usize) -> Self {
|
|
|
|
self.rs_quorum = Some(quorum);
|
|
|
|
self
|
|
|
|
}
|
|
|
|
/// Set timeout of the strategy
|
|
|
|
pub fn with_timeout(mut self, timeout: Duration) -> Self {
|
|
|
|
self.rs_timeout = timeout;
|
|
|
|
self
|
|
|
|
}
|
|
|
|
/// Set if requests can be dropped after quorum has been reached
|
|
|
|
/// In general true for read requests, and false for write
|
|
|
|
pub fn interrupt_after_quorum(mut self, interrupt: bool) -> Self {
|
|
|
|
self.rs_interrupt_after_quorum = interrupt;
|
|
|
|
self
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
#[derive(Clone)]
|
2021-11-04 15:04:26 +00:00
|
|
|
pub struct RpcHelper(Arc<RpcHelperInner>);
|
|
|
|
|
|
|
|
struct RpcHelperInner {
|
|
|
|
our_node_id: Uuid,
|
|
|
|
fullmesh: Arc<FullMeshPeeringStrategy>,
|
|
|
|
background: Arc<BackgroundRunner>,
|
|
|
|
ring: watch::Receiver<Arc<Ring>>,
|
2022-02-16 13:23:04 +00:00
|
|
|
request_buffer_semaphore: Arc<Semaphore>,
|
|
|
|
metrics: RpcMetrics,
|
2021-10-14 09:50:12 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
impl RpcHelper {
|
2021-11-03 16:00:40 +00:00
|
|
|
pub(crate) fn new(
|
2021-11-04 15:04:26 +00:00
|
|
|
our_node_id: Uuid,
|
2021-11-03 16:00:40 +00:00
|
|
|
fullmesh: Arc<FullMeshPeeringStrategy>,
|
|
|
|
background: Arc<BackgroundRunner>,
|
2021-11-04 15:04:26 +00:00
|
|
|
ring: watch::Receiver<Arc<Ring>>,
|
2021-11-03 16:00:40 +00:00
|
|
|
) -> Self {
|
2022-02-16 13:23:04 +00:00
|
|
|
let sem = Arc::new(Semaphore::new(REQUEST_BUFFER_SIZE));
|
|
|
|
|
|
|
|
let metrics = RpcMetrics::new(sem.clone());
|
|
|
|
|
2021-11-04 15:04:26 +00:00
|
|
|
Self(Arc::new(RpcHelperInner {
|
|
|
|
our_node_id,
|
2021-11-03 16:00:40 +00:00
|
|
|
fullmesh,
|
|
|
|
background,
|
2021-11-04 15:04:26 +00:00
|
|
|
ring,
|
2022-02-16 13:23:04 +00:00
|
|
|
request_buffer_semaphore: sem,
|
|
|
|
metrics,
|
2021-11-04 15:04:26 +00:00
|
|
|
}))
|
2021-11-03 16:00:40 +00:00
|
|
|
}
|
|
|
|
|
2021-10-15 09:05:09 +00:00
|
|
|
pub async fn call<M, H, S>(
|
2021-10-14 09:50:12 +00:00
|
|
|
&self,
|
|
|
|
endpoint: &Endpoint<M, H>,
|
2021-10-15 09:05:09 +00:00
|
|
|
to: Uuid,
|
2021-10-14 09:50:12 +00:00
|
|
|
msg: M,
|
|
|
|
strat: RequestStrategy,
|
2021-10-15 09:05:09 +00:00
|
|
|
) -> Result<S, Error>
|
2021-10-14 09:50:12 +00:00
|
|
|
where
|
2021-10-15 09:05:09 +00:00
|
|
|
M: Rpc<Response = Result<S, Error>>,
|
2021-10-14 09:50:12 +00:00
|
|
|
H: EndpointHandler<M>,
|
|
|
|
{
|
|
|
|
self.call_arc(endpoint, to, Arc::new(msg), strat).await
|
|
|
|
}
|
|
|
|
|
2021-10-15 09:05:09 +00:00
|
|
|
pub async fn call_arc<M, H, S>(
|
2021-10-14 09:50:12 +00:00
|
|
|
&self,
|
|
|
|
endpoint: &Endpoint<M, H>,
|
2021-10-15 09:05:09 +00:00
|
|
|
to: Uuid,
|
2021-10-14 09:50:12 +00:00
|
|
|
msg: Arc<M>,
|
|
|
|
strat: RequestStrategy,
|
2021-10-15 09:05:09 +00:00
|
|
|
) -> Result<S, Error>
|
2021-10-14 09:50:12 +00:00
|
|
|
where
|
2021-10-15 09:05:09 +00:00
|
|
|
M: Rpc<Response = Result<S, Error>>,
|
2021-10-14 09:50:12 +00:00
|
|
|
H: EndpointHandler<M>,
|
|
|
|
{
|
2022-02-16 13:23:04 +00:00
|
|
|
let metric_tags = [KeyValue::new("endpoint", endpoint.path().to_string())];
|
|
|
|
|
2021-11-03 16:00:40 +00:00
|
|
|
let msg_size = rmp_to_vec_all_named(&msg)?.len() as u32;
|
2021-11-04 15:04:26 +00:00
|
|
|
let permit = self
|
|
|
|
.0
|
|
|
|
.request_buffer_semaphore
|
|
|
|
.acquire_many(msg_size)
|
2022-02-22 12:53:59 +00:00
|
|
|
.record_duration(&self.0.metrics.rpc_queueing_time, &metric_tags)
|
2021-11-04 15:04:26 +00:00
|
|
|
.await?;
|
2021-11-03 16:00:40 +00:00
|
|
|
|
2022-02-16 13:23:04 +00:00
|
|
|
self.0.metrics.rpc_counter.add(1, &metric_tags);
|
|
|
|
|
2021-10-15 09:05:09 +00:00
|
|
|
let node_id = to.into();
|
2022-02-22 12:53:59 +00:00
|
|
|
let rpc_call = endpoint.call(&node_id, msg, strat.rs_priority)
|
|
|
|
.record_duration(&self.0.metrics.rpc_duration, &metric_tags);
|
2022-02-17 22:28:23 +00:00
|
|
|
|
2021-10-14 09:50:12 +00:00
|
|
|
select! {
|
2022-02-17 22:28:23 +00:00
|
|
|
res = rpc_call => {
|
2021-11-03 16:00:40 +00:00
|
|
|
drop(permit);
|
2022-02-16 13:23:04 +00:00
|
|
|
|
|
|
|
if res.is_err() {
|
|
|
|
self.0.metrics.rpc_netapp_error_counter.add(1, &metric_tags);
|
|
|
|
}
|
|
|
|
let res = res?;
|
|
|
|
|
|
|
|
if res.is_err() {
|
|
|
|
self.0.metrics.rpc_garage_error_counter.add(1, &metric_tags);
|
|
|
|
}
|
|
|
|
|
|
|
|
Ok(res?)
|
2021-11-03 16:00:40 +00:00
|
|
|
}
|
|
|
|
_ = tokio::time::sleep(strat.rs_timeout) => {
|
|
|
|
drop(permit);
|
2022-02-16 13:23:04 +00:00
|
|
|
self.0.metrics.rpc_timeout_counter.add(1, &metric_tags);
|
2021-11-03 16:00:40 +00:00
|
|
|
Err(Error::Timeout)
|
|
|
|
}
|
2021-10-14 09:50:12 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2021-10-15 09:05:09 +00:00
|
|
|
pub async fn call_many<M, H, S>(
|
2021-10-14 09:50:12 +00:00
|
|
|
&self,
|
|
|
|
endpoint: &Endpoint<M, H>,
|
2021-10-15 09:05:09 +00:00
|
|
|
to: &[Uuid],
|
2021-10-14 09:50:12 +00:00
|
|
|
msg: M,
|
|
|
|
strat: RequestStrategy,
|
2021-10-15 09:05:09 +00:00
|
|
|
) -> Vec<(Uuid, Result<S, Error>)>
|
2021-10-14 09:50:12 +00:00
|
|
|
where
|
2021-10-15 09:05:09 +00:00
|
|
|
M: Rpc<Response = Result<S, Error>>,
|
2021-10-14 09:50:12 +00:00
|
|
|
H: EndpointHandler<M>,
|
|
|
|
{
|
|
|
|
let msg = Arc::new(msg);
|
|
|
|
let resps = join_all(
|
|
|
|
to.iter()
|
|
|
|
.map(|to| self.call_arc(endpoint, *to, msg.clone(), strat)),
|
|
|
|
)
|
|
|
|
.await;
|
|
|
|
to.iter()
|
|
|
|
.cloned()
|
|
|
|
.zip(resps.into_iter())
|
|
|
|
.collect::<Vec<_>>()
|
|
|
|
}
|
|
|
|
|
2021-10-15 09:05:09 +00:00
|
|
|
pub async fn broadcast<M, H, S>(
|
2021-10-14 09:50:12 +00:00
|
|
|
&self,
|
|
|
|
endpoint: &Endpoint<M, H>,
|
|
|
|
msg: M,
|
|
|
|
strat: RequestStrategy,
|
2021-10-15 09:05:09 +00:00
|
|
|
) -> Vec<(Uuid, Result<S, Error>)>
|
2021-10-14 09:50:12 +00:00
|
|
|
where
|
2021-10-15 09:05:09 +00:00
|
|
|
M: Rpc<Response = Result<S, Error>>,
|
2021-10-14 09:50:12 +00:00
|
|
|
H: EndpointHandler<M>,
|
|
|
|
{
|
|
|
|
let to = self
|
2021-11-04 15:04:26 +00:00
|
|
|
.0
|
2021-10-14 09:50:12 +00:00
|
|
|
.fullmesh
|
|
|
|
.get_peer_list()
|
|
|
|
.iter()
|
2021-10-15 09:05:09 +00:00
|
|
|
.map(|p| p.id.into())
|
2021-10-14 09:50:12 +00:00
|
|
|
.collect::<Vec<_>>();
|
|
|
|
self.call_many(endpoint, &to[..], msg, strat).await
|
|
|
|
}
|
|
|
|
|
2021-11-04 15:04:26 +00:00
|
|
|
/// Make a RPC call to multiple servers, returning either a Vec of responses,
|
|
|
|
/// or an error if quorum could not be reached due to too many errors
|
2021-10-15 09:05:09 +00:00
|
|
|
pub async fn try_call_many<M, H, S>(
|
2021-10-14 09:50:12 +00:00
|
|
|
&self,
|
|
|
|
endpoint: &Arc<Endpoint<M, H>>,
|
2021-10-15 09:05:09 +00:00
|
|
|
to: &[Uuid],
|
2021-10-14 09:50:12 +00:00
|
|
|
msg: M,
|
|
|
|
strategy: RequestStrategy,
|
2021-10-15 09:05:09 +00:00
|
|
|
) -> Result<Vec<S>, Error>
|
2021-10-14 09:50:12 +00:00
|
|
|
where
|
2021-10-15 09:05:09 +00:00
|
|
|
M: Rpc<Response = Result<S, Error>> + 'static,
|
2021-10-14 09:50:12 +00:00
|
|
|
H: EndpointHandler<M> + 'static,
|
2022-02-17 22:28:23 +00:00
|
|
|
S: Send + 'static,
|
2021-10-14 09:50:12 +00:00
|
|
|
{
|
2021-11-04 15:04:26 +00:00
|
|
|
let quorum = strategy.rs_quorum.unwrap_or(to.len());
|
2021-10-14 09:50:12 +00:00
|
|
|
|
2022-02-17 22:28:23 +00:00
|
|
|
let tracer = opentelemetry::global::tracer("garage");
|
2022-02-18 19:39:55 +00:00
|
|
|
let mut span = tracer.start(format!("RPC {} to {}", endpoint.path(), to.len()));
|
2022-02-17 22:28:23 +00:00
|
|
|
span.set_attribute(KeyValue::new("to", format!("{:?}", to)));
|
|
|
|
span.set_attribute(KeyValue::new("quorum", quorum as i64));
|
|
|
|
|
|
|
|
async {
|
|
|
|
let msg = Arc::new(msg);
|
|
|
|
|
|
|
|
// Build future for each request
|
|
|
|
// They are not started now: they are added below in a FuturesUnordered
|
|
|
|
// object that will take care of polling them (see below)
|
|
|
|
let requests = to.iter().cloned().map(|to| {
|
|
|
|
let self2 = self.clone();
|
|
|
|
let msg = msg.clone();
|
|
|
|
let endpoint2 = endpoint.clone();
|
|
|
|
(to, async move {
|
|
|
|
self2.call_arc(&endpoint2, to, msg, strategy).await
|
2021-11-04 15:04:26 +00:00
|
|
|
})
|
2022-02-17 22:28:23 +00:00
|
|
|
});
|
|
|
|
|
|
|
|
// Vectors in which success results and errors will be collected
|
|
|
|
let mut successes = vec![];
|
|
|
|
let mut errors = vec![];
|
|
|
|
|
|
|
|
if strategy.rs_interrupt_after_quorum {
|
|
|
|
// Case 1: once quorum is reached, other requests don't matter.
|
|
|
|
// What we do here is only send the required number of requests
|
|
|
|
// to reach a quorum, priorizing nodes with the lowest latency.
|
|
|
|
// When there are errors, we start new requests to compensate.
|
|
|
|
|
|
|
|
// Retrieve some status variables that we will use to sort requests
|
|
|
|
let peer_list = self.0.fullmesh.get_peer_list();
|
|
|
|
let ring: Arc<Ring> = self.0.ring.borrow().clone();
|
|
|
|
let our_zone = match ring.layout.node_role(&self.0.our_node_id) {
|
|
|
|
Some(pc) => &pc.zone,
|
|
|
|
None => "",
|
|
|
|
};
|
|
|
|
|
|
|
|
// Augment requests with some information used to sort them.
|
|
|
|
// The tuples are as follows:
|
|
|
|
// (is another node?, is another zone?, latency, node ID, request future)
|
|
|
|
// We store all of these tuples in a vec that we can sort.
|
|
|
|
// By sorting this vec, we priorize ourself, then nodes in the same zone,
|
|
|
|
// and within a same zone we priorize nodes with the lowest latency.
|
|
|
|
let mut requests = requests
|
|
|
|
.map(|(to, fut)| {
|
|
|
|
let peer_zone = match ring.layout.node_role(&to) {
|
|
|
|
Some(pc) => &pc.zone,
|
|
|
|
None => "",
|
|
|
|
};
|
|
|
|
let peer_avg_ping = peer_list
|
|
|
|
.iter()
|
|
|
|
.find(|x| x.id.as_ref() == to.as_slice())
|
|
|
|
.map(|pi| pi.avg_ping)
|
|
|
|
.flatten()
|
|
|
|
.unwrap_or_else(|| Duration::from_secs(1));
|
|
|
|
(
|
|
|
|
to != self.0.our_node_id,
|
|
|
|
peer_zone != our_zone,
|
|
|
|
peer_avg_ping,
|
|
|
|
to,
|
|
|
|
fut,
|
|
|
|
)
|
|
|
|
})
|
|
|
|
.collect::<Vec<_>>();
|
|
|
|
|
|
|
|
// Sort requests by (priorize ourself, priorize same zone, priorize low latency)
|
|
|
|
requests.sort_by_key(|(diffnode, diffzone, ping, _to, _fut)| {
|
|
|
|
(*diffnode, *diffzone, *ping)
|
|
|
|
});
|
2021-11-04 15:04:26 +00:00
|
|
|
|
2022-02-17 22:28:23 +00:00
|
|
|
// Make an iterator to take requests in their sorted order
|
|
|
|
let mut requests = requests.into_iter();
|
|
|
|
|
|
|
|
// resp_stream will contain all of the requests that are currently in flight.
|
|
|
|
// (for the moment none, they will be added in the loop below)
|
|
|
|
let mut resp_stream = FuturesUnordered::new();
|
|
|
|
|
|
|
|
// Do some requests and collect results
|
|
|
|
'request_loop: while successes.len() < quorum {
|
|
|
|
// If the current set of requests that are running is not enough to possibly
|
|
|
|
// reach quorum, start some new requests.
|
|
|
|
while successes.len() + resp_stream.len() < quorum {
|
|
|
|
if let Some((_, _, _, req_to, fut)) = requests.next() {
|
|
|
|
let span = tracer.start(format!("RPC to {:?}", req_to));
|
|
|
|
resp_stream.push(tokio::spawn(
|
|
|
|
fut.with_context(Context::current_with_span(span)),
|
|
|
|
));
|
|
|
|
} else {
|
|
|
|
// If we have no request to add, we know that we won't ever
|
|
|
|
// reach quorum: bail out now.
|
|
|
|
break 'request_loop;
|
|
|
|
}
|
2021-11-04 15:04:26 +00:00
|
|
|
}
|
2022-02-17 22:28:23 +00:00
|
|
|
assert!(!resp_stream.is_empty()); // because of loop invariants
|
|
|
|
|
|
|
|
// Wait for one request to terminate
|
|
|
|
match resp_stream.next().await.unwrap().unwrap() {
|
|
|
|
Ok(msg) => {
|
|
|
|
successes.push(msg);
|
|
|
|
}
|
|
|
|
Err(e) => {
|
|
|
|
errors.push(e);
|
|
|
|
}
|
2021-11-04 15:04:26 +00:00
|
|
|
}
|
2021-10-14 09:50:12 +00:00
|
|
|
}
|
2022-02-17 22:28:23 +00:00
|
|
|
} else {
|
|
|
|
// Case 2: all of the requests need to be sent in all cases,
|
|
|
|
// and need to terminate. (this is the case for writes that
|
|
|
|
// must be spread to n nodes)
|
|
|
|
// Just start all the requests in parallel and return as soon
|
|
|
|
// as the quorum is reached.
|
|
|
|
let mut resp_stream = requests
|
|
|
|
.map(|(_, fut)| fut)
|
|
|
|
.collect::<FuturesUnordered<_>>();
|
|
|
|
|
|
|
|
while let Some(resp) = resp_stream.next().await {
|
|
|
|
match resp {
|
|
|
|
Ok(msg) => {
|
|
|
|
successes.push(msg);
|
|
|
|
if successes.len() >= quorum {
|
|
|
|
break;
|
|
|
|
}
|
|
|
|
}
|
|
|
|
Err(e) => {
|
|
|
|
errors.push(e);
|
2021-11-04 15:04:26 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2022-02-17 22:28:23 +00:00
|
|
|
if !resp_stream.is_empty() {
|
|
|
|
// Continue remaining requests in background.
|
|
|
|
// Continue the remaining requests immediately using tokio::spawn
|
|
|
|
// but enqueue a task in the background runner
|
|
|
|
// to ensure that the process won't exit until the requests are done
|
|
|
|
// (if we had just enqueued the resp_stream.collect directly in the background runner,
|
|
|
|
// the requests might have been put on hold in the background runner's queue,
|
|
|
|
// in which case they might timeout or otherwise fail)
|
|
|
|
let wait_finished_fut = tokio::spawn(async move {
|
|
|
|
resp_stream.collect::<Vec<Result<_, _>>>().await;
|
|
|
|
});
|
|
|
|
self.0.background.spawn(wait_finished_fut.map(|_| Ok(())));
|
|
|
|
}
|
2021-10-14 09:50:12 +00:00
|
|
|
}
|
|
|
|
|
2022-02-17 22:28:23 +00:00
|
|
|
if successes.len() >= quorum {
|
|
|
|
Ok(successes)
|
|
|
|
} else {
|
|
|
|
let errors = errors.iter().map(|e| format!("{}", e)).collect::<Vec<_>>();
|
|
|
|
Err(Error::Quorum(quorum, successes.len(), to.len(), errors))
|
|
|
|
}
|
2021-10-14 09:50:12 +00:00
|
|
|
}
|
2022-02-17 22:28:23 +00:00
|
|
|
.with_context(Context::current_with_span(span))
|
|
|
|
.await
|
2021-10-14 09:50:12 +00:00
|
|
|
}
|
|
|
|
}
|