pub struct MetaClient {
worker_id: u32,
worker_type: WorkerType,
host_addr: HostAddr,
inner: GrpcMetaClient,
meta_config: MetaConfig,
cluster_id: String,
shutting_down: Arc<AtomicBool>,
}
Expand description
Client to meta server. Cloning the instance is lightweight.
Fields§
§worker_id: u32
§worker_type: WorkerType
§host_addr: HostAddr
§inner: GrpcMetaClient
§meta_config: MetaConfig
§cluster_id: String
§shutting_down: Arc<AtomicBool>
Implementations§
source§impl MetaClient
impl MetaClient
pub fn worker_id(&self) -> u32
pub fn host_addr(&self) -> &HostAddr
pub fn worker_type(&self) -> WorkerType
pub fn cluster_id(&self) -> &str
sourcepub async fn subscribe(
&self,
subscribe_type: SubscribeType,
) -> Result<Streaming<SubscribeResponse>>
pub async fn subscribe( &self, subscribe_type: SubscribeType, ) -> Result<Streaming<SubscribeResponse>>
Subscribe to notification from meta.
pub async fn create_connection( &self, connection_name: String, database_id: u32, schema_id: u32, owner_id: u32, req: Payload, ) -> Result<WaitVersion>
pub async fn create_secret( &self, secret_name: String, database_id: u32, schema_id: u32, owner_id: u32, value: Vec<u8>, ) -> Result<WaitVersion>
pub async fn list_connections( &self, _name: Option<&str>, ) -> Result<Vec<Connection>>
pub async fn drop_connection(&self, connection_id: u32) -> Result<WaitVersion>
pub async fn drop_secret(&self, secret_id: SecretId) -> Result<WaitVersion>
sourcepub async fn register_new(
addr_strategy: MetaAddressStrategy,
worker_type: WorkerType,
addr: &HostAddr,
property: Property,
meta_config: &MetaConfig,
) -> (Self, SystemParamsReader)
pub async fn register_new( addr_strategy: MetaAddressStrategy, worker_type: WorkerType, addr: &HostAddr, property: Property, meta_config: &MetaConfig, ) -> (Self, SystemParamsReader)
Register the current node to the cluster and set the corresponding worker id.
Retry if there’s connection issue with the meta node. Exit the process if the registration fails.
async fn register_new_inner( addr_strategy: MetaAddressStrategy, worker_type: WorkerType, addr: &HostAddr, property: Property, meta_config: &MetaConfig, ) -> Result<(Self, SystemParamsReader)>
sourcepub async fn activate(&self, addr: &HostAddr) -> Result<()>
pub async fn activate(&self, addr: &HostAddr) -> Result<()>
Activate the current node in cluster to confirm it’s ready to serve.
sourcepub async fn send_heartbeat(&self, node_id: u32) -> Result<()>
pub async fn send_heartbeat(&self, node_id: u32) -> Result<()>
Send heartbeat signal to meta service.
pub async fn create_database(&self, db: PbDatabase) -> Result<WaitVersion>
pub async fn create_schema(&self, schema: PbSchema) -> Result<WaitVersion>
pub async fn create_materialized_view( &self, table: PbTable, graph: StreamFragmentGraph, dependencies: HashSet<ObjectId>, ) -> Result<WaitVersion>
pub async fn drop_materialized_view( &self, table_id: TableId, cascade: bool, ) -> Result<WaitVersion>
pub async fn create_source( &self, source: PbSource, graph: Option<StreamFragmentGraph>, ) -> Result<WaitVersion>
pub async fn create_sink( &self, sink: PbSink, graph: StreamFragmentGraph, affected_table_change: Option<ReplaceTablePlan>, dependencies: HashSet<ObjectId>, ) -> Result<WaitVersion>
pub async fn create_subscription( &self, subscription: PbSubscription, ) -> Result<WaitVersion>
pub async fn create_function(&self, function: PbFunction) -> Result<WaitVersion>
pub async fn create_table( &self, source: Option<PbSource>, table: PbTable, graph: StreamFragmentGraph, job_type: PbTableJobType, ) -> Result<WaitVersion>
pub async fn comment_on(&self, comment: PbComment) -> Result<WaitVersion>
pub async fn alter_name( &self, object: Object, name: &str, ) -> Result<WaitVersion>
pub async fn alter_owner( &self, object: Object, owner_id: u32, ) -> Result<WaitVersion>
pub async fn alter_set_schema( &self, object: Object, new_schema_id: u32, ) -> Result<WaitVersion>
pub async fn alter_source(&self, source: PbSource) -> Result<WaitVersion>
pub async fn alter_parallelism( &self, table_id: u32, parallelism: PbTableParallelism, deferred: bool, ) -> Result<()>
pub async fn alter_swap_rename(&self, object: Object) -> Result<WaitVersion>
pub async fn replace_table( &self, source: Option<PbSource>, table: PbTable, graph: StreamFragmentGraph, table_col_index_mapping: ColIndexMapping, job_type: PbTableJobType, ) -> Result<WaitVersion>
pub async fn auto_schema_change( &self, schema_change: SchemaChangeEnvelope, ) -> Result<()>
pub async fn create_view(&self, view: PbView) -> Result<WaitVersion>
pub async fn create_index( &self, index: PbIndex, table: PbTable, graph: StreamFragmentGraph, ) -> Result<WaitVersion>
pub async fn drop_table( &self, source_id: Option<u32>, table_id: TableId, cascade: bool, ) -> Result<WaitVersion>
pub async fn drop_view( &self, view_id: u32, cascade: bool, ) -> Result<WaitVersion>
pub async fn drop_source( &self, source_id: u32, cascade: bool, ) -> Result<WaitVersion>
pub async fn drop_sink( &self, sink_id: u32, cascade: bool, affected_table_change: Option<ReplaceTablePlan>, ) -> Result<WaitVersion>
pub async fn drop_subscription( &self, subscription_id: u32, cascade: bool, ) -> Result<WaitVersion>
pub async fn drop_index( &self, index_id: IndexId, cascade: bool, ) -> Result<WaitVersion>
pub async fn drop_function( &self, function_id: FunctionId, ) -> Result<WaitVersion>
pub async fn drop_database(&self, database_id: u32) -> Result<WaitVersion>
pub async fn drop_schema(&self, schema_id: u32) -> Result<WaitVersion>
pub async fn create_user(&self, user: UserInfo) -> Result<u64>
pub async fn drop_user(&self, user_id: u32) -> Result<u64>
pub async fn update_user( &self, user: UserInfo, update_fields: Vec<UpdateField>, ) -> Result<u64>
pub async fn grant_privilege( &self, user_ids: Vec<u32>, privileges: Vec<GrantPrivilege>, with_grant_option: bool, granted_by: u32, ) -> Result<u64>
pub async fn revoke_privilege( &self, user_ids: Vec<u32>, privileges: Vec<GrantPrivilege>, granted_by: u32, revoke_by: u32, revoke_grant_option: bool, cascade: bool, ) -> Result<u64>
sourcepub async fn unregister(&self) -> Result<()>
pub async fn unregister(&self) -> Result<()>
Unregister the current node from the cluster.
sourcepub async fn try_unregister(&self)
pub async fn try_unregister(&self)
Try to unregister the current worker from the cluster with best effort. Log the result.
pub async fn update_schedulability( &self, worker_ids: &[u32], schedulability: Schedulability, ) -> Result<UpdateWorkerNodeSchedulabilityResponse>
pub async fn list_worker_nodes( &self, worker_type: Option<WorkerType>, ) -> Result<Vec<WorkerNode>>
sourcepub fn start_heartbeat_loop(
meta_client: MetaClient,
min_interval: Duration,
) -> (JoinHandle<()>, Sender<()>)
pub fn start_heartbeat_loop( meta_client: MetaClient, min_interval: Duration, ) -> (JoinHandle<()>, Sender<()>)
Starts a heartbeat worker.
pub async fn risectl_list_state_tables(&self) -> Result<Vec<PbTable>>
pub async fn flush(&self, database_id: u32) -> Result<HummockVersionId>
pub async fn wait(&self) -> Result<()>
pub async fn recover(&self) -> Result<()>
pub async fn cancel_creating_jobs(&self, jobs: PbJobs) -> Result<Vec<u32>>
pub async fn list_table_fragments( &self, table_ids: &[u32], ) -> Result<HashMap<u32, TableFragmentInfo>>
pub async fn list_table_fragment_states( &self, ) -> Result<Vec<TableFragmentState>>
pub async fn list_fragment_distributions( &self, ) -> Result<Vec<FragmentDistribution>>
pub async fn list_actor_states(&self) -> Result<Vec<ActorState>>
pub async fn list_actor_splits(&self) -> Result<Vec<ActorSplit>>
pub async fn list_object_dependencies( &self, ) -> Result<Vec<PbObjectDependencies>>
pub async fn pause(&self) -> Result<PauseResponse>
pub async fn resume(&self) -> Result<ResumeResponse>
pub async fn apply_throttle( &self, kind: PbThrottleTarget, id: u32, rate: Option<u32>, ) -> Result<ApplyThrottleResponse>
pub async fn get_cluster_recovery_status(&self) -> Result<RecoveryStatus>
pub async fn get_cluster_info(&self) -> Result<GetClusterInfoResponse>
pub async fn reschedule( &self, worker_reschedules: HashMap<u32, PbWorkerReschedule>, revision: u64, resolve_no_shuffle_upstream: bool, ) -> Result<(bool, u64)>
pub async fn risectl_get_pinned_versions_summary( &self, ) -> Result<RiseCtlGetPinnedVersionsSummaryResponse>
pub async fn risectl_get_checkpoint_hummock_version( &self, ) -> Result<RiseCtlGetCheckpointVersionResponse>
pub async fn risectl_pause_hummock_version_checkpoint( &self, ) -> Result<RiseCtlPauseVersionCheckpointResponse>
pub async fn risectl_resume_hummock_version_checkpoint( &self, ) -> Result<RiseCtlResumeVersionCheckpointResponse>
pub async fn init_metadata_for_replay( &self, tables: Vec<PbTable>, compaction_groups: Vec<CompactionGroupInfo>, ) -> Result<()>
pub async fn replay_version_delta( &self, version_delta: HummockVersionDelta, ) -> Result<(HummockVersion, Vec<CompactionGroupId>)>
pub async fn list_version_deltas( &self, start_id: HummockVersionId, num_limit: u32, committed_epoch_limit: HummockEpoch, ) -> Result<Vec<HummockVersionDelta>>
pub async fn trigger_compaction_deterministic( &self, version_id: HummockVersionId, compaction_groups: Vec<CompactionGroupId>, ) -> Result<()>
pub async fn disable_commit_epoch(&self) -> Result<HummockVersion>
pub async fn get_assigned_compact_task_num(&self) -> Result<usize>
pub async fn risectl_list_compaction_group( &self, ) -> Result<Vec<CompactionGroupInfo>>
pub async fn risectl_update_compaction_config( &self, compaction_groups: &[CompactionGroupId], configs: &[MutableConfig], ) -> Result<()>
pub async fn backup_meta(&self, remarks: Option<String>) -> Result<u64>
pub async fn get_backup_job_status( &self, job_id: u64, ) -> Result<(BackupJobStatus, String)>
pub async fn delete_meta_snapshot(&self, snapshot_ids: &[u64]) -> Result<()>
pub async fn get_meta_snapshot_manifest(&self) -> Result<MetaSnapshotManifest>
pub async fn get_telemetry_info(&self) -> Result<TelemetryInfoResponse>
pub async fn get_system_params(&self) -> Result<SystemParamsReader>
pub async fn set_system_param( &self, param: String, value: Option<String>, ) -> Result<Option<SystemParamsReader>>
pub async fn get_session_params(&self) -> Result<String>
pub async fn set_session_param( &self, param: String, value: Option<String>, ) -> Result<String>
pub async fn get_ddl_progress(&self) -> Result<Vec<DdlProgress>>
pub async fn split_compaction_group( &self, group_id: CompactionGroupId, table_ids_to_new_group: &[StateTableId], partition_vnode_count: u32, ) -> Result<CompactionGroupId>
pub async fn get_tables(&self, table_ids: &[u32]) -> Result<HashMap<u32, Table>>
pub async fn list_serving_vnode_mappings( &self, ) -> Result<HashMap<u32, (u32, WorkerSlotMapping)>>
pub async fn risectl_list_compaction_status( &self, ) -> Result<(Vec<CompactStatus>, Vec<CompactTaskAssignment>, Vec<CompactTaskProgress>)>
pub async fn get_compaction_score( &self, compaction_group_id: CompactionGroupId, ) -> Result<Vec<PickerInfo>>
pub async fn risectl_rebuild_table_stats(&self) -> Result<()>
pub async fn list_branched_object(&self) -> Result<Vec<BranchedObject>>
pub async fn list_active_write_limit(&self) -> Result<HashMap<u64, WriteLimit>>
pub async fn list_hummock_meta_config(&self) -> Result<HashMap<String, String>>
pub async fn delete_worker_node(&self, worker: HostAddress) -> Result<()>
pub async fn rw_cloud_validate_source( &self, source_type: SourceType, source_config: HashMap<String, String>, ) -> Result<RwCloudValidateSourceResponse>
pub async fn sink_coordinate_client(&self) -> SinkCoordinationRpcClient
pub async fn list_compact_task_assignment( &self, ) -> Result<Vec<CompactTaskAssignment>>
pub async fn list_event_log(&self) -> Result<Vec<EventLog>>
pub async fn list_compact_task_progress( &self, ) -> Result<Vec<CompactTaskProgress>>
sourcepub fn try_add_panic_event_blocking(
&self,
panic_info: impl Display,
timeout_millis: Option<u64>,
)
pub fn try_add_panic_event_blocking( &self, panic_info: impl Display, timeout_millis: Option<u64>, )
If timeout_millis
is None, default is used.
pub async fn add_sink_fail_evet( &self, sink_id: u32, sink_name: String, connector: String, error: String, ) -> Result<()>
pub async fn cancel_compact_task( &self, task_id: u64, task_status: TaskStatus, ) -> Result<bool>
pub async fn get_version_by_epoch( &self, epoch: HummockEpoch, table_id: u32, ) -> Result<PbHummockVersion>
pub async fn get_cluster_limits(&self) -> Result<Vec<ClusterLimit>>
pub async fn merge_compaction_group( &self, left_group_id: CompactionGroupId, right_group_id: CompactionGroupId, ) -> Result<()>
sourcepub async fn list_rate_limits(&self) -> Result<Vec<RateLimitInfo>>
pub async fn list_rate_limits(&self) -> Result<Vec<RateLimitInfo>>
List all rate limits for sources and backfills
Trait Implementations§
source§impl Clone for MetaClient
impl Clone for MetaClient
source§fn clone(&self) -> MetaClient
fn clone(&self) -> MetaClient
Returns a copy of the value. Read more
1.0.0 · source§fn clone_from(&mut self, source: &Self)
fn clone_from(&mut self, source: &Self)
Performs copy-assignment from
source
. Read moresource§impl Debug for MetaClient
impl Debug for MetaClient
source§impl HummockMetaClient for MetaClient
impl HummockMetaClient for MetaClient
fn unpin_version_before<'life0, 'async_trait>(
&'life0 self,
unpin_version_before: HummockVersionId,
) -> Pin<Box<dyn Future<Output = Result<()>> + Send + 'async_trait>>where
Self: 'async_trait,
'life0: 'async_trait,
fn get_current_version<'life0, 'async_trait>(
&'life0 self,
) -> Pin<Box<dyn Future<Output = Result<HummockVersion>> + Send + 'async_trait>>where
Self: 'async_trait,
'life0: 'async_trait,
fn get_new_sst_ids<'life0, 'async_trait>(
&'life0 self,
number: u32,
) -> Pin<Box<dyn Future<Output = Result<SstObjectIdRange>> + Send + 'async_trait>>where
Self: 'async_trait,
'life0: 'async_trait,
fn commit_epoch<'life0, 'async_trait>(
&'life0 self,
_epoch: HummockEpoch,
_sync_result: SyncResult,
_is_log_store: bool,
) -> Pin<Box<dyn Future<Output = Result<()>> + Send + 'async_trait>>where
Self: 'async_trait,
'life0: 'async_trait,
fn trigger_manual_compaction<'life0, 'async_trait>(
&'life0 self,
compaction_group_id: u64,
table_id: u32,
level: u32,
sst_ids: Vec<u64>,
) -> Pin<Box<dyn Future<Output = Result<()>> + Send + 'async_trait>>where
Self: 'async_trait,
'life0: 'async_trait,
fn trigger_full_gc<'life0, 'async_trait>(
&'life0 self,
sst_retention_time_sec: u64,
prefix: Option<String>,
) -> Pin<Box<dyn Future<Output = Result<()>> + Send + 'async_trait>>where
Self: 'async_trait,
'life0: 'async_trait,
fn subscribe_compaction_event<'life0, 'async_trait>(
&'life0 self,
) -> Pin<Box<dyn Future<Output = Result<(UnboundedSender<SubscribeCompactionEventRequest>, BoxStream<'static, CompactionEventItem>)>> + Send + 'async_trait>>where
Self: 'async_trait,
'life0: 'async_trait,
fn get_version_by_epoch<'life0, 'async_trait>(
&'life0 self,
epoch: HummockEpoch,
table_id: u32,
) -> Pin<Box<dyn Future<Output = Result<PbHummockVersion>> + Send + 'async_trait>>where
Self: 'async_trait,
'life0: 'async_trait,
source§impl TelemetryInfoFetcher for MetaClient
impl TelemetryInfoFetcher for MetaClient
Auto Trait Implementations§
impl Freeze for MetaClient
impl !RefUnwindSafe for MetaClient
impl Send for MetaClient
impl Sync for MetaClient
impl Unpin for MetaClient
impl !UnwindSafe for MetaClient
Blanket Implementations§
source§impl<T> BorrowMut<T> for Twhere
T: ?Sized,
impl<T> BorrowMut<T> for Twhere
T: ?Sized,
source§fn borrow_mut(&mut self) -> &mut T
fn borrow_mut(&mut self) -> &mut T
Mutably borrows from an owned value. Read more
source§impl<T> CloneToUninit for Twhere
T: Clone,
impl<T> CloneToUninit for Twhere
T: Clone,
source§unsafe fn clone_to_uninit(&self, dst: *mut T)
unsafe fn clone_to_uninit(&self, dst: *mut T)
🔬This is a nightly-only experimental API. (
clone_to_uninit
)§impl<T> FutureExt for T
impl<T> FutureExt for T
§fn with_context(self, otel_cx: Context) -> WithContext<Self>
fn with_context(self, otel_cx: Context) -> WithContext<Self>
§fn with_current_context(self) -> WithContext<Self>
fn with_current_context(self) -> WithContext<Self>
§impl<T> Instrument for T
impl<T> Instrument for T
§fn instrument(self, span: Span) -> Instrumented<Self>
fn instrument(self, span: Span) -> Instrumented<Self>
§fn in_current_span(self) -> Instrumented<Self>
fn in_current_span(self) -> Instrumented<Self>
source§impl<T> Instrument for T
impl<T> Instrument for T
source§fn instrument(self, span: Span) -> Instrumented<Self>
fn instrument(self, span: Span) -> Instrumented<Self>
source§fn in_current_span(self) -> Instrumented<Self>
fn in_current_span(self) -> Instrumented<Self>
source§impl<T> IntoEither for T
impl<T> IntoEither for T
source§fn into_either(self, into_left: bool) -> Either<Self, Self>
fn into_either(self, into_left: bool) -> Either<Self, Self>
Converts
self
into a Left
variant of Either<Self, Self>
if into_left
is true
.
Converts self
into a Right
variant of Either<Self, Self>
otherwise. Read moresource§fn into_either_with<F>(self, into_left: F) -> Either<Self, Self>
fn into_either_with<F>(self, into_left: F) -> Either<Self, Self>
Converts
self
into a Left
variant of Either<Self, Self>
if into_left(&self)
returns true
.
Converts self
into a Right
variant of Either<Self, Self>
otherwise. Read moresource§impl<T> IntoRequest<T> for T
impl<T> IntoRequest<T> for T
source§fn into_request(self) -> Request<T>
fn into_request(self) -> Request<T>
Wrap the input message
T
in a tonic::Request
§impl<T> IntoResult<T> for T
impl<T> IntoResult<T> for T
type Err = Infallible
fn into_result(self) -> Result<T, <T as IntoResult<T>>::Err>
source§impl<M> MetricVecRelabelExt for M
impl<M> MetricVecRelabelExt for M
source§fn relabel(
self,
metric_level: MetricLevel,
relabel_threshold: MetricLevel,
) -> RelabeledMetricVec<M>
fn relabel( self, metric_level: MetricLevel, relabel_threshold: MetricLevel, ) -> RelabeledMetricVec<M>
Equivalent to
RelabeledMetricVec::with_metric_level
.source§fn relabel_n(
self,
metric_level: MetricLevel,
relabel_threshold: MetricLevel,
relabel_num: usize,
) -> RelabeledMetricVec<M>
fn relabel_n( self, metric_level: MetricLevel, relabel_threshold: MetricLevel, relabel_num: usize, ) -> RelabeledMetricVec<M>
Equivalent to
RelabeledMetricVec::with_metric_level_relabel_n
.source§fn relabel_debug_1(
self,
relabel_threshold: MetricLevel,
) -> RelabeledMetricVec<M>
fn relabel_debug_1( self, relabel_threshold: MetricLevel, ) -> RelabeledMetricVec<M>
Equivalent to
RelabeledMetricVec::with_metric_level_relabel_n
with metric_level
set to
MetricLevel::Debug
and relabel_num
set to 1.