Skip to content

Commit 2c7faec

Browse files
committed
Introduce a fast reconnect process for async cluster connections.
The process is periodic and can be configured via ClusterParams. This process ensures that all expected user connections exist and have not been passively closed. The expected connections are calculated from the current slot map. Additionally, for the Tokio runtime, an instant disconnect notification is available, allowing the reconnect process to be triggered instantly without waiting for the periodic check. This process is especially important for pub/sub support, as passive disconnects can render a pub/sub subscriber inoperative. Three integration tests are introduced with this feature: a generic fast reconnect test, pub/sub resilience to passive disconnects, and pub/sub resilience to scale-out.
1 parent 2d7200f commit 2c7faec

22 files changed

+1015
-628
lines changed

redis-test/src/lib.rs

+4
Original file line numberDiff line numberDiff line change
@@ -288,6 +288,10 @@ impl AioConnectionLike for MockRedisConnection {
288288
fn get_db(&self) -> i64 {
289289
0
290290
}
291+
292+
fn is_closed(&self) -> bool {
293+
false
294+
}
291295
}
292296

293297
#[cfg(test)]

redis/examples/async-await.rs

+1-1
Original file line numberDiff line numberDiff line change
@@ -4,7 +4,7 @@ use redis::AsyncCommands;
44
#[tokio::main]
55
async fn main() -> redis::RedisResult<()> {
66
let client = redis::Client::open("redis://127.0.0.1/").unwrap();
7-
let mut con = client.get_multiplexed_async_connection(None).await?;
7+
let mut con = client.get_multiplexed_async_connection(None, None).await?;
88

99
con.set("key1", b"foo").await?;
1010

redis/examples/async-connection-loss.rs

+3-1
Original file line numberDiff line numberDiff line change
@@ -80,7 +80,9 @@ async fn main() -> RedisResult<()> {
8080

8181
let client = redis::Client::open("redis://127.0.0.1/").unwrap();
8282
match mode {
83-
Mode::Default => run_multi(client.get_multiplexed_tokio_connection(None).await?).await?,
83+
Mode::Default => {
84+
run_multi(client.get_multiplexed_tokio_connection(None, None).await?).await?
85+
}
8486
Mode::Reconnect => run_multi(client.get_connection_manager().await?).await?,
8587
#[allow(deprecated)]
8688
Mode::Deprecated => run_single(client.get_async_connection(None).await?).await?,

redis/examples/async-multiplexed.rs

+4-1
Original file line numberDiff line numberDiff line change
@@ -34,7 +34,10 @@ async fn test_cmd(con: &MultiplexedConnection, i: i32) -> RedisResult<()> {
3434
async fn main() {
3535
let client = redis::Client::open("redis://127.0.0.1/").unwrap();
3636

37-
let con = client.get_multiplexed_tokio_connection(None).await.unwrap();
37+
let con = client
38+
.get_multiplexed_tokio_connection(None, None)
39+
.await
40+
.unwrap();
3841

3942
let cmds = (0..100).map(|i| test_cmd(&con, i));
4043
let result = future::try_join_all(cmds).await.unwrap();

redis/examples/async-pub-sub.rs

+1-1
Original file line numberDiff line numberDiff line change
@@ -5,7 +5,7 @@ use redis::AsyncCommands;
55
#[tokio::main]
66
async fn main() -> redis::RedisResult<()> {
77
let client = redis::Client::open("redis://127.0.0.1/").unwrap();
8-
let mut publish_conn = client.get_multiplexed_async_connection(None).await?;
8+
let mut publish_conn = client.get_multiplexed_async_connection(None, None).await?;
99
let mut pubsub_conn = client.get_async_pubsub().await?;
1010

1111
pubsub_conn.subscribe("wavephone").await?;

redis/examples/async-scan.rs

+1-1
Original file line numberDiff line numberDiff line change
@@ -5,7 +5,7 @@ use redis::{AsyncCommands, AsyncIter};
55
#[tokio::main]
66
async fn main() -> redis::RedisResult<()> {
77
let client = redis::Client::open("redis://127.0.0.1/").unwrap();
8-
let mut con = client.get_multiplexed_async_connection(None).await?;
8+
let mut con = client.get_multiplexed_async_connection(None, None).await?;
99

1010
con.set("async-key1", b"foo").await?;
1111
con.set("async-key2", b"foo").await?;

redis/src/aio/connection.rs

+5
Original file line numberDiff line numberDiff line change
@@ -305,6 +305,11 @@ where
305305
fn get_db(&self) -> i64 {
306306
self.db
307307
}
308+
309+
fn is_closed(&self) -> bool {
310+
// always false for AsyncRead + AsyncWrite (cant do better)
311+
false
312+
}
308313
}
309314

310315
/// Represents a `PubSub` connection.

redis/src/aio/connection_manager.rs

+6
Original file line numberDiff line numberDiff line change
@@ -196,6 +196,7 @@ impl ConnectionManager {
196196
response_timeout,
197197
connection_timeout,
198198
None,
199+
None,
199200
)
200201
})
201202
.await
@@ -301,4 +302,9 @@ impl ConnectionLike for ConnectionManager {
301302
fn get_db(&self) -> i64 {
302303
self.client.connection_info().redis.db
303304
}
305+
306+
fn is_closed(&self) -> bool {
307+
// always return false due to automatic reconnect
308+
false
309+
}
304310
}

redis/src/aio/mod.rs

+18
Original file line numberDiff line numberDiff line change
@@ -85,6 +85,24 @@ pub trait ConnectionLike {
8585
/// also might be incorrect if the connection like object is not
8686
/// actually connected.
8787
fn get_db(&self) -> i64;
88+
89+
/// Returns the state of the connection
90+
fn is_closed(&self) -> bool;
91+
}
92+
93+
/// Implements ability to notify about disconnection events
94+
pub trait DisconnectNotifier: Send + Sync {
95+
/// Notify about disconnect event
96+
fn notify_disconnect(&mut self);
97+
98+
/// Intended to be used with Box
99+
fn clone_box(&self) -> Box<dyn DisconnectNotifier>;
100+
}
101+
102+
impl Clone for Box<dyn DisconnectNotifier> {
103+
fn clone(&self) -> Box<dyn DisconnectNotifier> {
104+
self.clone_box()
105+
}
88106
}
89107

90108
// Initial setup for every connection.

redis/src/aio/multiplexed_connection.rs

+47-15
Original file line numberDiff line numberDiff line change
@@ -1,5 +1,6 @@
11
use super::{ConnectionLike, Runtime};
22
use crate::aio::setup_connection;
3+
use crate::aio::DisconnectNotifier;
34
use crate::cmd::Cmd;
45
#[cfg(any(feature = "tokio-comp", feature = "async-std-comp"))]
56
use crate::parser::ValueCodec;
@@ -23,6 +24,7 @@ use std::fmt;
2324
use std::fmt::Debug;
2425
use std::io;
2526
use std::pin::Pin;
27+
use std::sync::atomic::{AtomicBool, Ordering};
2628
use std::sync::Arc;
2729
use std::task::{self, Poll};
2830
use std::time::Duration;
@@ -73,19 +75,11 @@ struct PipelineMessage<S> {
7375
/// items being output by the `Stream` (the number is specified at time of sending). With the
7476
/// interface provided by `Pipeline` an easy interface of request to response, hiding the `Stream`
7577
/// and `Sink`.
78+
#[derive(Clone)]
7679
struct Pipeline<SinkItem> {
7780
sender: mpsc::Sender<PipelineMessage<SinkItem>>,
78-
7981
push_manager: Arc<ArcSwap<PushManager>>,
80-
}
81-
82-
impl<SinkItem> Clone for Pipeline<SinkItem> {
83-
fn clone(&self) -> Self {
84-
Pipeline {
85-
sender: self.sender.clone(),
86-
push_manager: self.push_manager.clone(),
87-
}
88-
}
82+
is_stream_closed: Arc<AtomicBool>,
8983
}
9084

9185
impl<SinkItem> Debug for Pipeline<SinkItem>
@@ -104,14 +98,21 @@ pin_project! {
10498
in_flight: VecDeque<InFlight>,
10599
error: Option<RedisError>,
106100
push_manager: Arc<ArcSwap<PushManager>>,
101+
disconnect_notifier: Option<Box<dyn DisconnectNotifier>>,
102+
is_stream_closed: Arc<AtomicBool>,
107103
}
108104
}
109105

110106
impl<T> PipelineSink<T>
111107
where
112108
T: Stream<Item = RedisResult<Value>> + 'static,
113109
{
114-
fn new<SinkItem>(sink_stream: T, push_manager: Arc<ArcSwap<PushManager>>) -> Self
110+
fn new<SinkItem>(
111+
sink_stream: T,
112+
push_manager: Arc<ArcSwap<PushManager>>,
113+
disconnect_notifier: Option<Box<dyn DisconnectNotifier>>,
114+
is_stream_closed: Arc<AtomicBool>,
115+
) -> Self
115116
where
116117
T: Sink<SinkItem, Error = RedisError> + Stream<Item = RedisResult<Value>> + 'static,
117118
{
@@ -120,6 +121,8 @@ where
120121
in_flight: VecDeque::new(),
121122
error: None,
122123
push_manager,
124+
disconnect_notifier,
125+
is_stream_closed,
123126
}
124127
}
125128

@@ -130,7 +133,15 @@ where
130133
Some(result) => result,
131134
// The redis response stream is not going to produce any more items so we `Err`
132135
// to break out of the `forward` combinator and stop handling requests
133-
None => return Poll::Ready(Err(())),
136+
None => {
137+
// this is the right place to notify about the passive TCP disconnect
138+
// In other places we cannot distinguish between the active destruction of MultiplexedConnection and passive disconnect
139+
if let Some(disconnect_notifier) = self.as_mut().project().disconnect_notifier {
140+
disconnect_notifier.notify_disconnect();
141+
}
142+
self.is_stream_closed.store(true, Ordering::Relaxed);
143+
return Poll::Ready(Err(()));
144+
}
134145
};
135146
self.as_mut().send_result(item);
136147
}
@@ -296,7 +307,10 @@ impl<SinkItem> Pipeline<SinkItem>
296307
where
297308
SinkItem: Send + 'static,
298309
{
299-
fn new<T>(sink_stream: T) -> (Self, impl Future<Output = ()>)
310+
fn new<T>(
311+
sink_stream: T,
312+
disconnect_notifier: Option<Box<dyn DisconnectNotifier>>,
313+
) -> (Self, impl Future<Output = ()>)
300314
where
301315
T: Sink<SinkItem, Error = RedisError> + Stream<Item = RedisResult<Value>> + 'static,
302316
T: Send + 'static,
@@ -308,7 +322,13 @@ where
308322
let (sender, mut receiver) = mpsc::channel(BUFFER_SIZE);
309323
let push_manager: Arc<ArcSwap<PushManager>> =
310324
Arc::new(ArcSwap::new(Arc::new(PushManager::default())));
311-
let sink = PipelineSink::new::<SinkItem>(sink_stream, push_manager.clone());
325+
let is_stream_closed = Arc::new(AtomicBool::new(false));
326+
let sink = PipelineSink::new::<SinkItem>(
327+
sink_stream,
328+
push_manager.clone(),
329+
disconnect_notifier,
330+
is_stream_closed.clone(),
331+
);
312332
let f = stream::poll_fn(move |cx| receiver.poll_recv(cx))
313333
.map(Ok)
314334
.forward(sink)
@@ -317,6 +337,7 @@ where
317337
Pipeline {
318338
sender,
319339
push_manager,
340+
is_stream_closed,
320341
},
321342
f,
322343
)
@@ -363,6 +384,10 @@ where
363384
async fn set_push_manager(&mut self, push_manager: PushManager) {
364385
self.push_manager.store(Arc::new(push_manager));
365386
}
387+
388+
pub fn is_closed(&self) -> bool {
389+
self.is_stream_closed.load(Ordering::Relaxed)
390+
}
366391
}
367392

368393
/// A connection object which can be cloned, allowing requests to be be sent concurrently
@@ -392,6 +417,7 @@ impl MultiplexedConnection {
392417
connection_info: &ConnectionInfo,
393418
stream: C,
394419
push_sender: Option<mpsc::UnboundedSender<PushInfo>>,
420+
disconnect_notifier: Option<Box<dyn DisconnectNotifier>>,
395421
) -> RedisResult<(Self, impl Future<Output = ()>)>
396422
where
397423
C: Unpin + AsyncRead + AsyncWrite + Send + 'static,
@@ -401,6 +427,7 @@ impl MultiplexedConnection {
401427
stream,
402428
std::time::Duration::MAX,
403429
push_sender,
430+
disconnect_notifier,
404431
)
405432
.await
406433
}
@@ -412,6 +439,7 @@ impl MultiplexedConnection {
412439
stream: C,
413440
response_timeout: std::time::Duration,
414441
push_sender: Option<mpsc::UnboundedSender<PushInfo>>,
442+
disconnect_notifier: Option<Box<dyn DisconnectNotifier>>,
415443
) -> RedisResult<(Self, impl Future<Output = ()>)>
416444
where
417445
C: Unpin + AsyncRead + AsyncWrite + Send + 'static,
@@ -429,7 +457,7 @@ impl MultiplexedConnection {
429457
let codec = ValueCodec::default()
430458
.framed(stream)
431459
.and_then(|msg| async move { msg });
432-
let (mut pipeline, driver) = Pipeline::new(codec);
460+
let (mut pipeline, driver) = Pipeline::new(codec, disconnect_notifier);
433461
let driver = boxed(driver);
434462
let pm = PushManager::default();
435463
if let Some(sender) = push_sender {
@@ -560,6 +588,10 @@ impl ConnectionLike for MultiplexedConnection {
560588
fn get_db(&self) -> i64 {
561589
self.db
562590
}
591+
592+
fn is_closed(&self) -> bool {
593+
self.pipeline.is_closed()
594+
}
563595
}
564596
impl MultiplexedConnection {
565597
/// Subscribes to a new channel.

0 commit comments

Comments
 (0)