mz_storage/
server.rs
1use std::sync::Arc;
13
14use mz_cluster::client::{ClusterClient, ClusterSpec};
15use mz_ore::metrics::MetricsRegistry;
16use mz_ore::now::NowFn;
17use mz_ore::tracing::TracingHandle;
18use mz_persist_client::cache::PersistClientCache;
19use mz_rocksdb::config::SharedWriteBufferManager;
20use mz_storage_client::client::{StorageClient, StorageCommand, StorageResponse};
21use mz_storage_types::connections::ConnectionContext;
22use mz_txn_wal::operator::TxnsContext;
23use timely::communication::Allocate;
24use timely::worker::Worker as TimelyWorker;
25use tokio::sync::mpsc;
26
27use crate::metrics::StorageMetrics;
28use crate::storage_state::{StorageInstanceContext, Worker};
29
30#[derive(Clone)]
32struct Config {
33 pub persist_clients: Arc<PersistClientCache>,
35 pub txns_ctx: TxnsContext,
37 pub tracing_handle: Arc<TracingHandle>,
39 pub now: NowFn,
41 pub connection_context: ConnectionContext,
43 pub instance_context: StorageInstanceContext,
45
46 pub metrics: StorageMetrics,
48 pub shared_rocksdb_write_buffer_manager: SharedWriteBufferManager,
50}
51
52pub fn serve(
54 metrics_registry: &MetricsRegistry,
55 persist_clients: Arc<PersistClientCache>,
56 txns_ctx: TxnsContext,
57 tracing_handle: Arc<TracingHandle>,
58 now: NowFn,
59 connection_context: ConnectionContext,
60 instance_context: StorageInstanceContext,
61) -> Result<impl Fn() -> Box<dyn StorageClient> + use<>, anyhow::Error> {
62 let config = Config {
63 persist_clients,
64 txns_ctx,
65 tracing_handle,
66 now,
67 connection_context,
68 instance_context,
69 metrics: StorageMetrics::register_with(metrics_registry),
70 shared_rocksdb_write_buffer_manager: Default::default(),
74 };
75 let tokio_executor = tokio::runtime::Handle::current();
76 let timely_container = Arc::new(tokio::sync::Mutex::new(None));
77
78 let client_builder = move || {
79 let client = ClusterClient::new(
80 Arc::clone(&timely_container),
81 tokio_executor.clone(),
82 config.clone(),
83 );
84 let client: Box<dyn StorageClient> = Box::new(client);
85 client
86 };
87
88 Ok(client_builder)
89}
90
91impl ClusterSpec for Config {
92 type Command = StorageCommand;
93 type Response = StorageResponse;
94
95 fn run_worker<A: Allocate + 'static>(
96 &self,
97 timely_worker: &mut TimelyWorker<A>,
98 client_rx: crossbeam_channel::Receiver<(
99 crossbeam_channel::Receiver<StorageCommand>,
100 mpsc::UnboundedSender<StorageResponse>,
101 )>,
102 ) {
103 Worker::new(
104 timely_worker,
105 client_rx,
106 self.metrics.clone(),
107 self.now.clone(),
108 self.connection_context.clone(),
109 self.instance_context.clone(),
110 Arc::clone(&self.persist_clients),
111 self.txns_ctx.clone(),
112 Arc::clone(&self.tracing_handle),
113 self.shared_rocksdb_write_buffer_manager.clone(),
114 )
115 .run();
116 }
117}