risingwave_rpc_client::meta_client

Struct MetaClient

source
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

source

pub fn worker_id(&self) -> u32

source

pub fn host_addr(&self) -> &HostAddr

source

pub fn worker_type(&self) -> WorkerType

source

pub fn cluster_id(&self) -> &str

source

pub async fn subscribe( &self, subscribe_type: SubscribeType, ) -> Result<Streaming<SubscribeResponse>>

Subscribe to notification from meta.

source

pub async fn create_connection( &self, connection_name: String, database_id: u32, schema_id: u32, owner_id: u32, req: Payload, ) -> Result<WaitVersion>

source

pub async fn create_secret( &self, secret_name: String, database_id: u32, schema_id: u32, owner_id: u32, value: Vec<u8>, ) -> Result<WaitVersion>

source

pub async fn list_connections( &self, _name: Option<&str>, ) -> Result<Vec<Connection>>

source

pub async fn drop_connection(&self, connection_id: u32) -> Result<WaitVersion>

source

pub async fn drop_secret(&self, secret_id: SecretId) -> Result<WaitVersion>

source

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.

source

async fn register_new_inner( addr_strategy: MetaAddressStrategy, worker_type: WorkerType, addr: &HostAddr, property: Property, meta_config: &MetaConfig, ) -> Result<(Self, SystemParamsReader)>

source

pub async fn activate(&self, addr: &HostAddr) -> Result<()>

Activate the current node in cluster to confirm it’s ready to serve.

source

pub async fn send_heartbeat(&self, node_id: u32) -> Result<()>

Send heartbeat signal to meta service.

source

pub async fn create_database(&self, db: PbDatabase) -> Result<WaitVersion>

source

pub async fn create_schema(&self, schema: PbSchema) -> Result<WaitVersion>

source

pub async fn create_materialized_view( &self, table: PbTable, graph: StreamFragmentGraph, dependencies: HashSet<ObjectId>, ) -> Result<WaitVersion>

source

pub async fn drop_materialized_view( &self, table_id: TableId, cascade: bool, ) -> Result<WaitVersion>

source

pub async fn create_source( &self, source: PbSource, graph: Option<StreamFragmentGraph>, ) -> Result<WaitVersion>

source

pub async fn create_sink( &self, sink: PbSink, graph: StreamFragmentGraph, affected_table_change: Option<ReplaceTablePlan>, dependencies: HashSet<ObjectId>, ) -> Result<WaitVersion>

source

pub async fn create_subscription( &self, subscription: PbSubscription, ) -> Result<WaitVersion>

source

pub async fn create_function(&self, function: PbFunction) -> Result<WaitVersion>

source

pub async fn create_table( &self, source: Option<PbSource>, table: PbTable, graph: StreamFragmentGraph, job_type: PbTableJobType, ) -> Result<WaitVersion>

source

pub async fn comment_on(&self, comment: PbComment) -> Result<WaitVersion>

source

pub async fn alter_name( &self, object: Object, name: &str, ) -> Result<WaitVersion>

source

pub async fn alter_owner( &self, object: Object, owner_id: u32, ) -> Result<WaitVersion>

source

pub async fn alter_set_schema( &self, object: Object, new_schema_id: u32, ) -> Result<WaitVersion>

source

pub async fn alter_source(&self, source: PbSource) -> Result<WaitVersion>

source

pub async fn alter_parallelism( &self, table_id: u32, parallelism: PbTableParallelism, deferred: bool, ) -> Result<()>

source

pub async fn alter_swap_rename(&self, object: Object) -> Result<WaitVersion>

source

pub async fn replace_table( &self, source: Option<PbSource>, table: PbTable, graph: StreamFragmentGraph, table_col_index_mapping: ColIndexMapping, job_type: PbTableJobType, ) -> Result<WaitVersion>

source

pub async fn auto_schema_change( &self, schema_change: SchemaChangeEnvelope, ) -> Result<()>

source

pub async fn create_view(&self, view: PbView) -> Result<WaitVersion>

source

pub async fn create_index( &self, index: PbIndex, table: PbTable, graph: StreamFragmentGraph, ) -> Result<WaitVersion>

source

pub async fn drop_table( &self, source_id: Option<u32>, table_id: TableId, cascade: bool, ) -> Result<WaitVersion>

source

pub async fn drop_view( &self, view_id: u32, cascade: bool, ) -> Result<WaitVersion>

source

pub async fn drop_source( &self, source_id: u32, cascade: bool, ) -> Result<WaitVersion>

source

pub async fn drop_sink( &self, sink_id: u32, cascade: bool, affected_table_change: Option<ReplaceTablePlan>, ) -> Result<WaitVersion>

source

pub async fn drop_subscription( &self, subscription_id: u32, cascade: bool, ) -> Result<WaitVersion>

source

pub async fn drop_index( &self, index_id: IndexId, cascade: bool, ) -> Result<WaitVersion>

source

pub async fn drop_function( &self, function_id: FunctionId, ) -> Result<WaitVersion>

source

pub async fn drop_database(&self, database_id: u32) -> Result<WaitVersion>

source

pub async fn drop_schema(&self, schema_id: u32) -> Result<WaitVersion>

source

pub async fn create_user(&self, user: UserInfo) -> Result<u64>

source

pub async fn drop_user(&self, user_id: u32) -> Result<u64>

source

pub async fn update_user( &self, user: UserInfo, update_fields: Vec<UpdateField>, ) -> Result<u64>

source

pub async fn grant_privilege( &self, user_ids: Vec<u32>, privileges: Vec<GrantPrivilege>, with_grant_option: bool, granted_by: u32, ) -> Result<u64>

source

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>

source

pub async fn unregister(&self) -> Result<()>

Unregister the current node from the cluster.

source

pub async fn try_unregister(&self)

Try to unregister the current worker from the cluster with best effort. Log the result.

source

pub async fn update_schedulability( &self, worker_ids: &[u32], schedulability: Schedulability, ) -> Result<UpdateWorkerNodeSchedulabilityResponse>

source

pub async fn list_worker_nodes( &self, worker_type: Option<WorkerType>, ) -> Result<Vec<WorkerNode>>

source

pub fn start_heartbeat_loop( meta_client: MetaClient, min_interval: Duration, ) -> (JoinHandle<()>, Sender<()>)

Starts a heartbeat worker.

source

pub async fn risectl_list_state_tables(&self) -> Result<Vec<PbTable>>

source

pub async fn flush(&self, database_id: u32) -> Result<HummockVersionId>

source

pub async fn wait(&self) -> Result<()>

source

pub async fn recover(&self) -> Result<()>

source

pub async fn cancel_creating_jobs(&self, jobs: PbJobs) -> Result<Vec<u32>>

source

pub async fn list_table_fragments( &self, table_ids: &[u32], ) -> Result<HashMap<u32, TableFragmentInfo>>

source

pub async fn list_table_fragment_states( &self, ) -> Result<Vec<TableFragmentState>>

source

pub async fn list_fragment_distributions( &self, ) -> Result<Vec<FragmentDistribution>>

source

pub async fn list_actor_states(&self) -> Result<Vec<ActorState>>

source

pub async fn list_actor_splits(&self) -> Result<Vec<ActorSplit>>

source

pub async fn list_object_dependencies( &self, ) -> Result<Vec<PbObjectDependencies>>

source

pub async fn pause(&self) -> Result<PauseResponse>

source

pub async fn resume(&self) -> Result<ResumeResponse>

source

pub async fn apply_throttle( &self, kind: PbThrottleTarget, id: u32, rate: Option<u32>, ) -> Result<ApplyThrottleResponse>

source

pub async fn get_cluster_recovery_status(&self) -> Result<RecoveryStatus>

source

pub async fn get_cluster_info(&self) -> Result<GetClusterInfoResponse>

source

pub async fn reschedule( &self, worker_reschedules: HashMap<u32, PbWorkerReschedule>, revision: u64, resolve_no_shuffle_upstream: bool, ) -> Result<(bool, u64)>

source

pub async fn risectl_get_pinned_versions_summary( &self, ) -> Result<RiseCtlGetPinnedVersionsSummaryResponse>

source

pub async fn risectl_get_checkpoint_hummock_version( &self, ) -> Result<RiseCtlGetCheckpointVersionResponse>

source

pub async fn risectl_pause_hummock_version_checkpoint( &self, ) -> Result<RiseCtlPauseVersionCheckpointResponse>

source

pub async fn risectl_resume_hummock_version_checkpoint( &self, ) -> Result<RiseCtlResumeVersionCheckpointResponse>

source

pub async fn init_metadata_for_replay( &self, tables: Vec<PbTable>, compaction_groups: Vec<CompactionGroupInfo>, ) -> Result<()>

source

pub async fn replay_version_delta( &self, version_delta: HummockVersionDelta, ) -> Result<(HummockVersion, Vec<CompactionGroupId>)>

source

pub async fn list_version_deltas( &self, start_id: HummockVersionId, num_limit: u32, committed_epoch_limit: HummockEpoch, ) -> Result<Vec<HummockVersionDelta>>

source

pub async fn trigger_compaction_deterministic( &self, version_id: HummockVersionId, compaction_groups: Vec<CompactionGroupId>, ) -> Result<()>

source

pub async fn disable_commit_epoch(&self) -> Result<HummockVersion>

source

pub async fn get_assigned_compact_task_num(&self) -> Result<usize>

source

pub async fn risectl_list_compaction_group( &self, ) -> Result<Vec<CompactionGroupInfo>>

source

pub async fn risectl_update_compaction_config( &self, compaction_groups: &[CompactionGroupId], configs: &[MutableConfig], ) -> Result<()>

source

pub async fn backup_meta(&self, remarks: Option<String>) -> Result<u64>

source

pub async fn get_backup_job_status( &self, job_id: u64, ) -> Result<(BackupJobStatus, String)>

source

pub async fn delete_meta_snapshot(&self, snapshot_ids: &[u64]) -> Result<()>

source

pub async fn get_meta_snapshot_manifest(&self) -> Result<MetaSnapshotManifest>

source

pub async fn get_telemetry_info(&self) -> Result<TelemetryInfoResponse>

source

pub async fn get_system_params(&self) -> Result<SystemParamsReader>

source

pub async fn set_system_param( &self, param: String, value: Option<String>, ) -> Result<Option<SystemParamsReader>>

source

pub async fn get_session_params(&self) -> Result<String>

source

pub async fn set_session_param( &self, param: String, value: Option<String>, ) -> Result<String>

source

pub async fn get_ddl_progress(&self) -> Result<Vec<DdlProgress>>

source

pub async fn split_compaction_group( &self, group_id: CompactionGroupId, table_ids_to_new_group: &[StateTableId], partition_vnode_count: u32, ) -> Result<CompactionGroupId>

source

pub async fn get_tables(&self, table_ids: &[u32]) -> Result<HashMap<u32, Table>>

source

pub async fn list_serving_vnode_mappings( &self, ) -> Result<HashMap<u32, (u32, WorkerSlotMapping)>>

source

pub async fn risectl_list_compaction_status( &self, ) -> Result<(Vec<CompactStatus>, Vec<CompactTaskAssignment>, Vec<CompactTaskProgress>)>

source

pub async fn get_compaction_score( &self, compaction_group_id: CompactionGroupId, ) -> Result<Vec<PickerInfo>>

source

pub async fn risectl_rebuild_table_stats(&self) -> Result<()>

source

pub async fn list_branched_object(&self) -> Result<Vec<BranchedObject>>

source

pub async fn list_active_write_limit(&self) -> Result<HashMap<u64, WriteLimit>>

source

pub async fn list_hummock_meta_config(&self) -> Result<HashMap<String, String>>

source

pub async fn delete_worker_node(&self, worker: HostAddress) -> Result<()>

source

pub async fn rw_cloud_validate_source( &self, source_type: SourceType, source_config: HashMap<String, String>, ) -> Result<RwCloudValidateSourceResponse>

source

pub async fn sink_coordinate_client(&self) -> SinkCoordinationRpcClient

source

pub async fn list_compact_task_assignment( &self, ) -> Result<Vec<CompactTaskAssignment>>

source

pub async fn list_event_log(&self) -> Result<Vec<EventLog>>

source

pub async fn list_compact_task_progress( &self, ) -> Result<Vec<CompactTaskProgress>>

source

pub fn try_add_panic_event_blocking( &self, panic_info: impl Display, timeout_millis: Option<u64>, )

If timeout_millis is None, default is used.

source

pub async fn add_sink_fail_evet( &self, sink_id: u32, sink_name: String, connector: String, error: String, ) -> Result<()>

source

pub async fn cancel_compact_task( &self, task_id: u64, task_status: TaskStatus, ) -> Result<bool>

source

pub async fn get_version_by_epoch( &self, epoch: HummockEpoch, table_id: u32, ) -> Result<PbHummockVersion>

source

pub async fn get_cluster_limits(&self) -> Result<Vec<ClusterLimit>>

source

pub async fn merge_compaction_group( &self, left_group_id: CompactionGroupId, right_group_id: CompactionGroupId, ) -> Result<()>

source

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

source§

fn clone(&self) -> MetaClient

Returns a copy of the value. Read more
1.0.0 · source§

fn clone_from(&mut self, source: &Self)

Performs copy-assignment from source. Read more
source§

impl Debug for MetaClient

source§

fn fmt(&self, f: &mut Formatter<'_>) -> Result

Formats the value using the given formatter. Read more
source§

impl HummockMetaClient for MetaClient

source§

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,

source§

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,

source§

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,

source§

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,

source§

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,

source§

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,

source§

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,

source§

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

source§

fn fetch_telemetry_info<'life0, 'async_trait>( &'life0 self, ) -> Pin<Box<dyn Future<Output = Result<Option<String>, String>> + Send + 'async_trait>>
where Self: 'async_trait, 'life0: 'async_trait,

Fetches telemetry info from meta. Currently it’s only tracking_id (cluster_id).

Auto Trait Implementations§

Blanket Implementations§

source§

impl<T> Any for T
where T: 'static + ?Sized,

source§

fn type_id(&self) -> TypeId

Gets the TypeId of self. Read more
source§

impl<T> Borrow<T> for T
where T: ?Sized,

source§

fn borrow(&self) -> &T

Immutably borrows from an owned value. Read more
source§

impl<T> BorrowMut<T> for T
where T: ?Sized,

source§

fn borrow_mut(&mut self) -> &mut T

Mutably borrows from an owned value. Read more
source§

impl<T> CloneToUninit for T
where T: Clone,

source§

unsafe fn clone_to_uninit(&self, dst: *mut T)

🔬This is a nightly-only experimental API. (clone_to_uninit)
Performs copy-assignment from self to dst. Read more
source§

impl<T> From<T> for T

source§

fn from(t: T) -> T

Returns the argument unchanged.

§

impl<T> FromRef<T> for T
where T: Clone,

§

fn from_ref(input: &T) -> T

Converts to this type from a reference to the input type.
§

impl<T> FutureExt for T

§

fn with_context(self, otel_cx: Context) -> WithContext<Self>

Attaches the provided Context to this type, returning a WithContext wrapper. Read more
§

fn with_current_context(self) -> WithContext<Self>

Attaches the current Context to this type, returning a WithContext wrapper. Read more
§

impl<T> Instrument for T

§

fn instrument(self, span: Span) -> Instrumented<Self>

Instruments this type with the provided [Span], returning an Instrumented wrapper. Read more
§

fn in_current_span(self) -> Instrumented<Self>

Instruments this type with the current Span, returning an Instrumented wrapper. Read more
source§

impl<T> Instrument for T

source§

fn instrument(self, span: Span) -> Instrumented<Self>

Instruments this type with the provided Span, returning an Instrumented wrapper. Read more
source§

fn in_current_span(self) -> Instrumented<Self>

Instruments this type with the current Span, returning an Instrumented wrapper. Read more
source§

impl<T, U> Into<U> for T
where U: From<T>,

source§

fn into(self) -> U

Calls U::from(self).

That is, this conversion is whatever the implementation of From<T> for U chooses to do.

source§

impl<T> IntoEither for T

source§

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 more
source§

fn into_either_with<F>(self, into_left: F) -> Either<Self, Self>
where F: FnOnce(&Self) -> bool,

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 more
source§

impl<T> IntoRequest<T> for T

source§

fn into_request(self) -> Request<T>

Wrap the input message T in a tonic::Request
§

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

source§

fn relabel( self, metric_level: MetricLevel, relabel_threshold: MetricLevel, ) -> RelabeledMetricVec<M>

source§

fn relabel_n( self, metric_level: MetricLevel, relabel_threshold: MetricLevel, relabel_num: usize, ) -> RelabeledMetricVec<M>

source§

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.
§

impl<T> Pointable for T

§

const ALIGN: usize = _

The alignment of pointer.
§

type Init = T

The type for initializers.
§

unsafe fn init(init: <T as Pointable>::Init) -> usize

Initializes a with the given initializer. Read more
§

unsafe fn deref<'a>(ptr: usize) -> &'a T

Dereferences the given pointer. Read more
§

unsafe fn deref_mut<'a>(ptr: usize) -> &'a mut T

Mutably dereferences the given pointer. Read more
§

unsafe fn drop(ptr: usize)

Drops the object pointed to by the given pointer. Read more
source§

impl<T> Same for T

source§

type Output = T

Should always be Self
source§

impl<T> ToOwned for T
where T: Clone,

source§

type Owned = T

The resulting type after obtaining ownership.
source§

fn to_owned(&self) -> T

Creates owned data from borrowed data, usually by cloning. Read more
source§

fn clone_into(&self, target: &mut T)

Uses borrowed data to replace owned data, usually by cloning. Read more
source§

impl<T, U> TryFrom<U> for T
where U: Into<T>,

source§

type Error = Infallible

The type returned in the event of a conversion error.
source§

fn try_from(value: U) -> Result<T, <T as TryFrom<U>>::Error>

Performs the conversion.
source§

impl<T, U> TryInto<U> for T
where U: TryFrom<T>,

source§

type Error = <U as TryFrom<T>>::Error

The type returned in the event of a conversion error.
source§

fn try_into(self) -> Result<U, <U as TryFrom<T>>::Error>

Performs the conversion.
§

impl<V, T> VZip<V> for T
where V: MultiLane<T>,

§

fn vzip(self) -> V

§

impl<T> WithSubscriber for T

§

fn with_subscriber<S>(self, subscriber: S) -> WithDispatch<Self>
where S: Into<Dispatch>,

Attaches the provided Subscriber to this type, returning a [WithDispatch] wrapper. Read more
§

fn with_current_subscriber(self) -> WithDispatch<Self>

Attaches the current default Subscriber to this type, returning a [WithDispatch] wrapper. Read more
source§

impl<T> WithSubscriber for T

source§

fn with_subscriber<S>(self, subscriber: S) -> WithDispatch<Self>
where S: Into<Dispatch>,

Attaches the provided Subscriber to this type, returning a WithDispatch wrapper. Read more
source§

fn with_current_subscriber(self) -> WithDispatch<Self>

Attaches the current default Subscriber to this type, returning a WithDispatch wrapper. Read more
§

impl<T> DevConfig for T
where T: Send + Sync + 'static + Debug,

source§

impl<T> LruValue for T
where T: Send + Sync,

§

impl<T> Value for T
where T: Send + Sync + 'static,