Documentation ¶
Index ¶
- Constants
- Variables
- func CheckOperatorValid(op *Operator) bool
- func CreateMergeRegionOperator(desc string, cluster Cluster, source *core.RegionInfo, target *core.RegionInfo, ...) (*Operator, *Operator, error)
- func DistinctScore(labels []string, stores []*core.StoreInfo, other *core.StoreInfo) float64
- func FilterSource(opt Options, store *core.StoreInfo, filters []Filter) bool
- func FilterTarget(opt Options, store *core.StoreInfo, filters []Filter) bool
- func RegisterScheduler(name string, createFn CreateSchedulerFunc)
- type AddLearner
- type AddPeer
- type BalanceSelector
- type BasicCluster
- func (bc *BasicCluster) BlockStore(storeID uint64) error
- func (bc *BasicCluster) CheckReadStatus(region *core.RegionInfo) (bool, *core.RegionStat)
- func (bc *BasicCluster) CheckWriteStatus(region *core.RegionInfo) (bool, *core.RegionStat)
- func (bc *BasicCluster) DeleteStore(store *core.StoreInfo)
- func (bc *BasicCluster) GetAdjacentRegions(region *core.RegionInfo) (*core.RegionInfo, *core.RegionInfo)
- func (bc *BasicCluster) GetAverageRegionSize() int64
- func (bc *BasicCluster) GetFollowerStores(region *core.RegionInfo) []*core.StoreInfo
- func (bc *BasicCluster) GetLeaderStore(region *core.RegionInfo) *core.StoreInfo
- func (bc *BasicCluster) GetRegion(regionID uint64) *core.RegionInfo
- func (bc *BasicCluster) GetRegionStores(region *core.RegionInfo) []*core.StoreInfo
- func (bc *BasicCluster) GetStore(storeID uint64) *core.StoreInfo
- func (bc *BasicCluster) GetStores() []*core.StoreInfo
- func (bc *BasicCluster) IsRegionHot(id uint64, hotThreshold int) bool
- func (bc *BasicCluster) PutRegion(region *core.RegionInfo) error
- func (bc *BasicCluster) PutStore(store *core.StoreInfo) error
- func (bc *BasicCluster) RandFollowerRegion(storeID uint64, opts ...core.RegionOption) *core.RegionInfo
- func (bc *BasicCluster) RandLeaderRegion(storeID uint64, opts ...core.RegionOption) *core.RegionInfo
- func (bc *BasicCluster) RegionReadStats() []*core.RegionStat
- func (bc *BasicCluster) RegionWriteStats() []*core.RegionStat
- func (bc *BasicCluster) UnblockStore(storeID uint64)
- type Cluster
- type CreateSchedulerFunc
- type Filter
- func NewBlockFilter() Filter
- func NewCacheFilter(cache *cache.TTLUint64) Filter
- func NewDisconnectFilter() Filter
- func NewDistinctScoreFilter(labels []string, stores []*core.StoreInfo, source *core.StoreInfo) Filter
- func NewExcludedFilter(sources, targets map[uint64]struct{}) Filter
- func NewHealthFilter() Filter
- func NewNamespaceFilter(classifier namespace.Classifier, namespace string) Filter
- func NewPendingPeerCountFilter() Filter
- func NewRejectLeaderFilter() Filter
- func NewSnapshotCountFilter() Filter
- func NewStateFilter() Filter
- func NewStorageThresholdFilter() Filter
- type FlowKind
- type HotSpotCache
- func (w *HotSpotCache) CheckRead(region *core.RegionInfo, stores *core.StoresInfo) (bool, *core.RegionStat)
- func (w *HotSpotCache) CheckWrite(region *core.RegionInfo, stores *core.StoresInfo) (bool, *core.RegionStat)
- func (w *HotSpotCache) CollectMetrics(stores *core.StoresInfo)
- func (w *HotSpotCache) RandHotRegionFromStore(storeID uint64, kind FlowKind, hotThreshold int) *core.RegionStat
- func (w *HotSpotCache) RegionStats(kind FlowKind) []*core.RegionStat
- func (w *HotSpotCache) Update(key uint64, item *core.RegionStat, kind FlowKind)
- type Limiter
- type MergeChecker
- type MergeRegion
- type MockCluster
- func (mc *MockCluster) AddLabelsStore(storeID uint64, regionCount int, labels map[string]string)
- func (mc *MockCluster) AddLeaderRegion(regionID uint64, leaderID uint64, followerIds ...uint64)
- func (mc *MockCluster) AddLeaderRegionWithRange(regionID uint64, startKey string, endKey string, leaderID uint64, ...)
- func (mc *MockCluster) AddLeaderRegionWithReadInfo(regionID uint64, leaderID uint64, readBytes uint64, followerIds ...uint64)
- func (mc *MockCluster) AddLeaderRegionWithWriteInfo(regionID uint64, leaderID uint64, writtenBytes uint64, followerIds ...uint64)
- func (mc *MockCluster) AddLeaderStore(storeID uint64, leaderCount int)
- func (mc *MockCluster) AddRegionStore(storeID uint64, regionCount int)
- func (mc *MockCluster) AllocPeer(storeID uint64) (*metapb.Peer, error)
- func (mc *MockCluster) ApplyOperator(op *Operator)
- func (mc *MockCluster) ApplyOperatorStep(region *core.RegionInfo, op *Operator) *core.RegionInfo
- func (mc *MockCluster) CheckLabelProperty(typ string, labels []*metapb.StoreLabel) bool
- func (mc *MockCluster) GetHotRegionScheduleLimit() uint64
- func (mc *MockCluster) GetLeaderScheduleLimit() uint64
- func (mc *MockCluster) GetMaxReplicas() int
- func (mc *MockCluster) GetMergeScheduleLimit() uint64
- func (mc *MockCluster) GetOpt() NamespaceOptions
- func (mc *MockCluster) GetRegionScheduleLimit() uint64
- func (mc *MockCluster) GetReplicaScheduleLimit() uint64
- func (mc *MockCluster) IsRegionHot(id uint64) bool
- func (mc *MockCluster) LoadRegion(regionID uint64, followerIds ...uint64)
- func (mc *MockCluster) RandHotRegionFromStore(store uint64, kind FlowKind) *core.RegionInfo
- func (mc *MockCluster) ScanRegions(startKey []byte, limit int) []*core.RegionInfo
- func (mc *MockCluster) SetStoreBusy(storeID uint64, busy bool)
- func (mc *MockCluster) SetStoreDisconnect(storeID uint64)
- func (mc *MockCluster) SetStoreDown(storeID uint64)
- func (mc *MockCluster) SetStoreOffline(storeID uint64)
- func (mc *MockCluster) SetStoreUp(storeID uint64)
- func (mc *MockCluster) UpdateLeaderCount(storeID uint64, leaderCount int)
- func (mc *MockCluster) UpdatePendingPeerCount(storeID uint64, pendingPeerCount int)
- func (mc *MockCluster) UpdateRegionCount(storeID uint64, regionCount int)
- func (mc *MockCluster) UpdateSnapshotCount(storeID uint64, snapshotCount int)
- func (mc *MockCluster) UpdateStorageRatio(storeID uint64, usedRatio, availableRatio float64)
- func (mc *MockCluster) UpdateStorageReadBytes(storeID uint64, BytesRead uint64)
- func (mc *MockCluster) UpdateStorageWrittenBytes(storeID uint64, BytesWritten uint64)
- func (mc *MockCluster) UpdateStoreLeaderSize(storeID uint64, size int64)
- func (mc *MockCluster) UpdateStoreLeaderWeight(storeID uint64, weight float64)
- func (mc *MockCluster) UpdateStoreRegionSize(storeID uint64, size int64)
- func (mc *MockCluster) UpdateStoreRegionWeight(storeID uint64, weight float64)
- func (mc *MockCluster) UpdateStoreStatus(id uint64)
- type MockSchedulerOptions
- func (mso *MockSchedulerOptions) GetHighSpaceRatio() float64
- func (mso *MockSchedulerOptions) GetHotRegionCacheHitsThreshold() int
- func (mso *MockSchedulerOptions) GetHotRegionScheduleLimit(name string) uint64
- func (mso *MockSchedulerOptions) GetLeaderScheduleLimit(name string) uint64
- func (mso *MockSchedulerOptions) GetLocationLabels() []string
- func (mso *MockSchedulerOptions) GetLowSpaceRatio() float64
- func (mso *MockSchedulerOptions) GetMaxMergeRegionKeys() uint64
- func (mso *MockSchedulerOptions) GetMaxMergeRegionSize() uint64
- func (mso *MockSchedulerOptions) GetMaxPendingPeerCount() uint64
- func (mso *MockSchedulerOptions) GetMaxReplicas(name string) int
- func (mso *MockSchedulerOptions) GetMaxSnapshotCount() uint64
- func (mso *MockSchedulerOptions) GetMaxStoreDownTime() time.Duration
- func (mso *MockSchedulerOptions) GetMergeScheduleLimit(name string) uint64
- func (mso *MockSchedulerOptions) GetRegionScheduleLimit(name string) uint64
- func (mso *MockSchedulerOptions) GetReplicaScheduleLimit(name string) uint64
- func (mso *MockSchedulerOptions) GetSplitMergeInterval() time.Duration
- func (mso *MockSchedulerOptions) GetTolerantSizeRatio() float64
- func (mso *MockSchedulerOptions) IsLocationReplacementEnabled() bool
- func (mso *MockSchedulerOptions) IsMakeUpReplicaEnabled() bool
- func (mso *MockSchedulerOptions) IsNamespaceRelocationEnabled() bool
- func (mso *MockSchedulerOptions) IsRaftLearnerEnabled() bool
- func (mso *MockSchedulerOptions) IsRemoveDownReplicaEnabled() bool
- func (mso *MockSchedulerOptions) IsRemoveExtraReplicaEnabled() bool
- func (mso *MockSchedulerOptions) IsReplaceOfflineReplicaEnabled() bool
- func (mso *MockSchedulerOptions) SetMaxReplicas(replicas int)
- type NamespaceChecker
- type NamespaceOptions
- type OpInfluence
- type Operator
- func CreateMovePeerOperator(desc string, cluster Cluster, region *core.RegionInfo, kind OperatorKind, ...) (*Operator, error)
- func CreateRemovePeerOperator(desc string, cluster Cluster, kind OperatorKind, region *core.RegionInfo, ...) (*Operator, error)
- func NewOperator(desc string, regionID uint64, regionEpoch *metapb.RegionEpoch, ...) *Operator
- func (o *Operator) AttachKind(kind OperatorKind)
- func (o *Operator) Check(region *core.RegionInfo) OperatorStep
- func (o *Operator) Desc() string
- func (o *Operator) ElapsedTime() time.Duration
- func (o *Operator) GetPriorityLevel() core.PriorityLevel
- func (o *Operator) History() []OperatorHistory
- func (o *Operator) Influence(opInfluence OpInfluence, region *core.RegionInfo)
- func (o *Operator) IsFinish() bool
- func (o *Operator) IsTimeout() bool
- func (o *Operator) Kind() OperatorKind
- func (o *Operator) Len() int
- func (o *Operator) MarshalJSON() ([]byte, error)
- func (o *Operator) RegionEpoch() *metapb.RegionEpoch
- func (o *Operator) RegionID() uint64
- func (o *Operator) SetDesc(desc string)
- func (o *Operator) SetPriorityLevel(level core.PriorityLevel)
- func (o *Operator) Step(i int) OperatorStep
- func (o *Operator) String() string
- type OperatorHistory
- type OperatorKind
- type OperatorStep
- type Options
- type PromoteLearner
- type RandomSelector
- type RangeCluster
- func (r *RangeCluster) GetAverageRegionSize() int64
- func (r *RangeCluster) GetFollowerStores(region *core.RegionInfo) []*core.StoreInfo
- func (r *RangeCluster) GetLeaderStore(region *core.RegionInfo) *core.StoreInfo
- func (r *RangeCluster) GetRegionStores(region *core.RegionInfo) []*core.StoreInfo
- func (r *RangeCluster) GetStore(id uint64) *core.StoreInfo
- func (r *RangeCluster) GetStores() []*core.StoreInfo
- func (r *RangeCluster) GetTolerantSizeRatio() float64
- func (r *RangeCluster) RandFollowerRegion(storeID uint64, opts ...core.RegionOption) *core.RegionInfo
- func (r *RangeCluster) RandLeaderRegion(storeID uint64, opts ...core.RegionOption) *core.RegionInfo
- func (r *RangeCluster) SetTolerantSizeRatio(ratio float64)
- type RegionScatterer
- type RemovePeer
- type ReplicaChecker
- type ReplicaSelector
- type Scheduler
- type SplitRegion
- type StoreInfluence
- type StoreStateFilter
- type TransferLeader
Constants ¶
const ( // LeaderOperatorWaitTime is the duration that when a leader operator lives // longer than it, the operator will be considered timeout. LeaderOperatorWaitTime = 10 * time.Second // RegionOperatorWaitTime is the duration that when a region operator lives // longer than it, the operator will be considered timeout. RegionOperatorWaitTime = 10 * time.Minute )
const (
// RegionHeartBeatReportInterval is the heartbeat report interval of a region
RegionHeartBeatReportInterval = 60
)
const ( // RejectLeader is the label property type that sugguests a store should not // have any region leaders. RejectLeader = "reject-leader" )
Variables ¶
var Simulating bool
Simulating is an option to overpass the impact of accelerated time. Should only turned on by the simulator.
Functions ¶
func CheckOperatorValid ¶
CheckOperatorValid checks if the operator is valid.
func CreateMergeRegionOperator ¶
func CreateMergeRegionOperator(desc string, cluster Cluster, source *core.RegionInfo, target *core.RegionInfo, kind OperatorKind) (*Operator, *Operator, error)
CreateMergeRegionOperator creates an Operator that merge two region into one
func DistinctScore ¶
DistinctScore returns the score that the other is distinct from the stores. A higher score means the other store is more different from the existed stores.
func FilterSource ¶
FilterSource checks if store can pass all Filters as source store.
func FilterTarget ¶
FilterTarget checks if store can pass all Filters as target store.
func RegisterScheduler ¶
func RegisterScheduler(name string, createFn CreateSchedulerFunc)
RegisterScheduler binds a scheduler creator. It should be called in init() func of a package.
Types ¶
type AddLearner ¶
type AddLearner struct {
ToStore, PeerID uint64
}
AddLearner is an OperatorStep that adds a region learner peer.
func (AddLearner) Influence ¶
func (al AddLearner) Influence(opInfluence OpInfluence, region *core.RegionInfo)
Influence calculates the store difference that current step make
func (AddLearner) IsFinish ¶
func (al AddLearner) IsFinish(region *core.RegionInfo) bool
IsFinish checks if current step is finished.
func (AddLearner) String ¶
func (al AddLearner) String() string
type AddPeer ¶
type AddPeer struct {
ToStore, PeerID uint64
}
AddPeer is an OperatorStep that adds a region peer.
func (AddPeer) Influence ¶
func (ap AddPeer) Influence(opInfluence OpInfluence, region *core.RegionInfo)
Influence calculates the store difference that current step make
func (AddPeer) IsFinish ¶
func (ap AddPeer) IsFinish(region *core.RegionInfo) bool
IsFinish checks if current step is finished.
type BalanceSelector ¶
type BalanceSelector struct {
// contains filtered or unexported fields
}
BalanceSelector selects source/target from store candidates based on their resource scores.
func NewBalanceSelector ¶
func NewBalanceSelector(kind core.ResourceKind, filters []Filter) *BalanceSelector
NewBalanceSelector creates a BalanceSelector instance.
func (*BalanceSelector) SelectSource ¶
SelectSource selects the store that can pass all filters and has the minimal resource score.
type BasicCluster ¶
type BasicCluster struct { Stores *core.StoresInfo Regions *core.RegionsInfo HotCache *HotSpotCache }
BasicCluster provides basic data member and interface for a tikv cluster.
func (*BasicCluster) BlockStore ¶
func (bc *BasicCluster) BlockStore(storeID uint64) error
BlockStore stops balancer from selecting the store.
func (*BasicCluster) CheckReadStatus ¶
func (bc *BasicCluster) CheckReadStatus(region *core.RegionInfo) (bool, *core.RegionStat)
CheckReadStatus checks the read status, returns whether need update statistics and item.
func (*BasicCluster) CheckWriteStatus ¶
func (bc *BasicCluster) CheckWriteStatus(region *core.RegionInfo) (bool, *core.RegionStat)
CheckWriteStatus checks the write status, returns whether need update statistics and item.
func (*BasicCluster) DeleteStore ¶
func (bc *BasicCluster) DeleteStore(store *core.StoreInfo)
DeleteStore deletes a store
func (*BasicCluster) GetAdjacentRegions ¶
func (bc *BasicCluster) GetAdjacentRegions(region *core.RegionInfo) (*core.RegionInfo, *core.RegionInfo)
GetAdjacentRegions returns region's info that is adjacent with specific region
func (*BasicCluster) GetAverageRegionSize ¶
func (bc *BasicCluster) GetAverageRegionSize() int64
GetAverageRegionSize returns the average region approximate size.
func (*BasicCluster) GetFollowerStores ¶
func (bc *BasicCluster) GetFollowerStores(region *core.RegionInfo) []*core.StoreInfo
GetFollowerStores returns all Stores that contains the region's follower peer.
func (*BasicCluster) GetLeaderStore ¶
func (bc *BasicCluster) GetLeaderStore(region *core.RegionInfo) *core.StoreInfo
GetLeaderStore returns all Stores that contains the region's leader peer.
func (*BasicCluster) GetRegion ¶
func (bc *BasicCluster) GetRegion(regionID uint64) *core.RegionInfo
GetRegion searches for a region by ID.
func (*BasicCluster) GetRegionStores ¶
func (bc *BasicCluster) GetRegionStores(region *core.RegionInfo) []*core.StoreInfo
GetRegionStores returns all Stores that contains the region's peer.
func (*BasicCluster) GetStore ¶
func (bc *BasicCluster) GetStore(storeID uint64) *core.StoreInfo
GetStore searches for a store by ID.
func (*BasicCluster) GetStores ¶
func (bc *BasicCluster) GetStores() []*core.StoreInfo
GetStores returns all Stores in the cluster.
func (*BasicCluster) IsRegionHot ¶
func (bc *BasicCluster) IsRegionHot(id uint64, hotThreshold int) bool
IsRegionHot checks if a region is in hot state.
func (*BasicCluster) PutRegion ¶
func (bc *BasicCluster) PutRegion(region *core.RegionInfo) error
PutRegion put a region
func (*BasicCluster) PutStore ¶
func (bc *BasicCluster) PutStore(store *core.StoreInfo) error
PutStore put a store
func (*BasicCluster) RandFollowerRegion ¶
func (bc *BasicCluster) RandFollowerRegion(storeID uint64, opts ...core.RegionOption) *core.RegionInfo
RandFollowerRegion returns a random region that has a follower on the store.
func (*BasicCluster) RandLeaderRegion ¶
func (bc *BasicCluster) RandLeaderRegion(storeID uint64, opts ...core.RegionOption) *core.RegionInfo
RandLeaderRegion returns a random region that has leader on the store.
func (*BasicCluster) RegionReadStats ¶
func (bc *BasicCluster) RegionReadStats() []*core.RegionStat
RegionReadStats returns hot region's read stats.
func (*BasicCluster) RegionWriteStats ¶
func (bc *BasicCluster) RegionWriteStats() []*core.RegionStat
RegionWriteStats returns hot region's write stats.
func (*BasicCluster) UnblockStore ¶
func (bc *BasicCluster) UnblockStore(storeID uint64)
UnblockStore allows balancer to select the store.
type Cluster ¶
type Cluster interface { RandFollowerRegion(storeID uint64, opts ...core.RegionOption) *core.RegionInfo RandLeaderRegion(storeID uint64, opts ...core.RegionOption) *core.RegionInfo GetAverageRegionSize() int64 GetStores() []*core.StoreInfo GetStore(id uint64) *core.StoreInfo GetRegion(id uint64) *core.RegionInfo GetRegionStores(region *core.RegionInfo) []*core.StoreInfo GetFollowerStores(region *core.RegionInfo) []*core.StoreInfo GetLeaderStore(region *core.RegionInfo) *core.StoreInfo GetAdjacentRegions(region *core.RegionInfo) (*core.RegionInfo, *core.RegionInfo) ScanRegions(startKey []byte, limit int) []*core.RegionInfo BlockStore(id uint64) error UnblockStore(id uint64) IsRegionHot(id uint64) bool RegionWriteStats() []*core.RegionStat RegionReadStats() []*core.RegionStat RandHotRegionFromStore(store uint64, kind FlowKind) *core.RegionInfo // get config methods GetOpt() NamespaceOptions Options // TODO: it should be removed. Schedulers don't need to know anything // about peers. AllocPeer(storeID uint64) (*metapb.Peer, error) }
Cluster provides an overview of a cluster's regions distribution.
type CreateSchedulerFunc ¶
CreateSchedulerFunc is for creating scheudler.
type Filter ¶
type Filter interface { Type() string // Return true if the store should not be used as a source store. FilterSource(opt Options, store *core.StoreInfo) bool // Return true if the store should not be used as a target store. FilterTarget(opt Options, store *core.StoreInfo) bool }
Filter is an interface to filter source and target store.
func NewBlockFilter ¶
func NewBlockFilter() Filter
NewBlockFilter creates a Filter that filters all stores that are blocked from balance.
func NewCacheFilter ¶
NewCacheFilter creates a Filter that filters all stores that are in the cache.
func NewDisconnectFilter ¶
func NewDisconnectFilter() Filter
NewDisconnectFilter creates a Filter that filters all stores that are disconnected.
func NewDistinctScoreFilter ¶
func NewDistinctScoreFilter(labels []string, stores []*core.StoreInfo, source *core.StoreInfo) Filter
NewDistinctScoreFilter creates a filter that filters all stores that have lower distinct score than specified store.
func NewExcludedFilter ¶
NewExcludedFilter creates a Filter that filters all specified stores.
func NewHealthFilter ¶
func NewHealthFilter() Filter
NewHealthFilter creates a Filter that filters all stores that are Busy or Down.
func NewNamespaceFilter ¶
func NewNamespaceFilter(classifier namespace.Classifier, namespace string) Filter
NewNamespaceFilter creates a Filter that filters all stores that are not belong to a namespace.
func NewPendingPeerCountFilter ¶
func NewPendingPeerCountFilter() Filter
NewPendingPeerCountFilter creates a Filter that filters all stores that are currently handling too many pending peers.
func NewRejectLeaderFilter ¶
func NewRejectLeaderFilter() Filter
NewRejectLeaderFilter creates a Filter that filters stores that marked as rejectLeader from being the target of leader transfer.
func NewSnapshotCountFilter ¶
func NewSnapshotCountFilter() Filter
NewSnapshotCountFilter creates a Filter that filters all stores that are currently handling too many snapshots.
type HotSpotCache ¶
type HotSpotCache struct {
// contains filtered or unexported fields
}
HotSpotCache is a cache hold hot regions.
func (*HotSpotCache) CheckRead ¶
func (w *HotSpotCache) CheckRead(region *core.RegionInfo, stores *core.StoresInfo) (bool, *core.RegionStat)
CheckRead checks the read status, returns whether need update statistics and item.
func (*HotSpotCache) CheckWrite ¶
func (w *HotSpotCache) CheckWrite(region *core.RegionInfo, stores *core.StoresInfo) (bool, *core.RegionStat)
CheckWrite checks the write status, returns whether need update statistics and item.
func (*HotSpotCache) CollectMetrics ¶
func (w *HotSpotCache) CollectMetrics(stores *core.StoresInfo)
CollectMetrics collect the hot cache metrics
func (*HotSpotCache) RandHotRegionFromStore ¶
func (w *HotSpotCache) RandHotRegionFromStore(storeID uint64, kind FlowKind, hotThreshold int) *core.RegionStat
RandHotRegionFromStore random picks a hot region in specify store.
func (*HotSpotCache) RegionStats ¶
func (w *HotSpotCache) RegionStats(kind FlowKind) []*core.RegionStat
RegionStats returns hot items according to kind
func (*HotSpotCache) Update ¶
func (w *HotSpotCache) Update(key uint64, item *core.RegionStat, kind FlowKind)
Update updates the cache.
type Limiter ¶
Limiter a counter that limits the number of operators
func (*Limiter) OperatorCount ¶
func (l *Limiter) OperatorCount(mask OperatorKind) uint64
OperatorCount gets the count of operators filtered by mask.
type MergeChecker ¶
type MergeChecker struct {
// contains filtered or unexported fields
}
MergeChecker ensures region to merge with adjacent region when size is small
func NewMergeChecker ¶
func NewMergeChecker(cluster Cluster, classifier namespace.Classifier) *MergeChecker
NewMergeChecker creates a merge checker.
func (*MergeChecker) Check ¶
func (m *MergeChecker) Check(region *core.RegionInfo) (*Operator, *Operator)
Check verifies a region's replicas, creating an Operator if need.
func (*MergeChecker) RecordRegionSplit ¶
func (m *MergeChecker) RecordRegionSplit(regionID uint64)
RecordRegionSplit put the recently splitted region into cache. MergeChecker will skip check it for a while.
type MergeRegion ¶
type MergeRegion struct { FromRegion *metapb.Region ToRegion *metapb.Region // there are two regions involved in merge process, // so to keep them from other scheduler, // both of them should add MerRegion operatorStep. // But actually, tikv just need the region want to be merged to get the merge request, // thus use a IsPssive mark to indicate that // this region doesn't need to send merge request to tikv. IsPassive bool }
MergeRegion is an OperatorStep that merge two regions.
func (MergeRegion) Influence ¶
func (mr MergeRegion) Influence(opInfluence OpInfluence, region *core.RegionInfo)
Influence calculates the store difference that current step make
func (MergeRegion) IsFinish ¶
func (mr MergeRegion) IsFinish(region *core.RegionInfo) bool
IsFinish checks if current step is finished
func (MergeRegion) String ¶
func (mr MergeRegion) String() string
type MockCluster ¶
type MockCluster struct { *BasicCluster *MockSchedulerOptions // contains filtered or unexported fields }
MockCluster is used to mock clusterInfo for test use
func NewMockCluster ¶
func NewMockCluster(opt *MockSchedulerOptions) *MockCluster
NewMockCluster creates a new MockCluster
func (*MockCluster) AddLabelsStore ¶
func (mc *MockCluster) AddLabelsStore(storeID uint64, regionCount int, labels map[string]string)
AddLabelsStore adds store with specified count of region and labels.
func (*MockCluster) AddLeaderRegion ¶
func (mc *MockCluster) AddLeaderRegion(regionID uint64, leaderID uint64, followerIds ...uint64)
AddLeaderRegion adds region with specified leader and followers.
func (*MockCluster) AddLeaderRegionWithRange ¶
func (mc *MockCluster) AddLeaderRegionWithRange(regionID uint64, startKey string, endKey string, leaderID uint64, followerIds ...uint64)
AddLeaderRegionWithRange adds region with specified leader, followers and key range.
func (*MockCluster) AddLeaderRegionWithReadInfo ¶
func (mc *MockCluster) AddLeaderRegionWithReadInfo(regionID uint64, leaderID uint64, readBytes uint64, followerIds ...uint64)
AddLeaderRegionWithReadInfo adds region with specified leader, followers and read info.
func (*MockCluster) AddLeaderRegionWithWriteInfo ¶
func (mc *MockCluster) AddLeaderRegionWithWriteInfo(regionID uint64, leaderID uint64, writtenBytes uint64, followerIds ...uint64)
AddLeaderRegionWithWriteInfo adds region with specified leader, followers and write info.
func (*MockCluster) AddLeaderStore ¶
func (mc *MockCluster) AddLeaderStore(storeID uint64, leaderCount int)
AddLeaderStore adds store with specified count of leader.
func (*MockCluster) AddRegionStore ¶
func (mc *MockCluster) AddRegionStore(storeID uint64, regionCount int)
AddRegionStore adds store with specified count of region.
func (*MockCluster) AllocPeer ¶
func (mc *MockCluster) AllocPeer(storeID uint64) (*metapb.Peer, error)
AllocPeer allocs a new peer on a store.
func (*MockCluster) ApplyOperator ¶
func (mc *MockCluster) ApplyOperator(op *Operator)
ApplyOperator mocks apply operator.
func (*MockCluster) ApplyOperatorStep ¶
func (mc *MockCluster) ApplyOperatorStep(region *core.RegionInfo, op *Operator) *core.RegionInfo
ApplyOperatorStep mocks apply operator step.
func (*MockCluster) CheckLabelProperty ¶
func (mc *MockCluster) CheckLabelProperty(typ string, labels []*metapb.StoreLabel) bool
CheckLabelProperty checks label property.
func (*MockCluster) GetHotRegionScheduleLimit ¶
func (mc *MockCluster) GetHotRegionScheduleLimit() uint64
GetHotRegionScheduleLimit mocks method.
func (*MockCluster) GetLeaderScheduleLimit ¶
func (mc *MockCluster) GetLeaderScheduleLimit() uint64
GetLeaderScheduleLimit mocks method.
func (*MockCluster) GetMaxReplicas ¶
func (mc *MockCluster) GetMaxReplicas() int
GetMaxReplicas mocks method.
func (*MockCluster) GetMergeScheduleLimit ¶
func (mc *MockCluster) GetMergeScheduleLimit() uint64
GetMergeScheduleLimit mocks method.
func (*MockCluster) GetRegionScheduleLimit ¶
func (mc *MockCluster) GetRegionScheduleLimit() uint64
GetRegionScheduleLimit mocks method.
func (*MockCluster) GetReplicaScheduleLimit ¶
func (mc *MockCluster) GetReplicaScheduleLimit() uint64
GetReplicaScheduleLimit mocks method.
func (*MockCluster) IsRegionHot ¶
func (mc *MockCluster) IsRegionHot(id uint64) bool
IsRegionHot checks if the region is hot
func (*MockCluster) LoadRegion ¶
func (mc *MockCluster) LoadRegion(regionID uint64, followerIds ...uint64)
LoadRegion put region info without leader
func (*MockCluster) RandHotRegionFromStore ¶
func (mc *MockCluster) RandHotRegionFromStore(store uint64, kind FlowKind) *core.RegionInfo
RandHotRegionFromStore random picks a hot region in specify store.
func (*MockCluster) ScanRegions ¶
func (mc *MockCluster) ScanRegions(startKey []byte, limit int) []*core.RegionInfo
ScanRegions scan region with start key, until number greater than limit.
func (*MockCluster) SetStoreBusy ¶
func (mc *MockCluster) SetStoreBusy(storeID uint64, busy bool)
SetStoreBusy sets store busy.
func (*MockCluster) SetStoreDisconnect ¶
func (mc *MockCluster) SetStoreDisconnect(storeID uint64)
SetStoreDisconnect changes a store's state to disconnected.
func (*MockCluster) SetStoreDown ¶
func (mc *MockCluster) SetStoreDown(storeID uint64)
SetStoreDown sets store down.
func (*MockCluster) SetStoreOffline ¶
func (mc *MockCluster) SetStoreOffline(storeID uint64)
SetStoreOffline sets store state to be offline.
func (*MockCluster) SetStoreUp ¶
func (mc *MockCluster) SetStoreUp(storeID uint64)
SetStoreUp sets store state to be up.
func (*MockCluster) UpdateLeaderCount ¶
func (mc *MockCluster) UpdateLeaderCount(storeID uint64, leaderCount int)
UpdateLeaderCount updates store leader count.
func (*MockCluster) UpdatePendingPeerCount ¶
func (mc *MockCluster) UpdatePendingPeerCount(storeID uint64, pendingPeerCount int)
UpdatePendingPeerCount updates store pending peer count.
func (*MockCluster) UpdateRegionCount ¶
func (mc *MockCluster) UpdateRegionCount(storeID uint64, regionCount int)
UpdateRegionCount updates store region count.
func (*MockCluster) UpdateSnapshotCount ¶
func (mc *MockCluster) UpdateSnapshotCount(storeID uint64, snapshotCount int)
UpdateSnapshotCount updates store snapshot count.
func (*MockCluster) UpdateStorageRatio ¶
func (mc *MockCluster) UpdateStorageRatio(storeID uint64, usedRatio, availableRatio float64)
UpdateStorageRatio updates store storage ratio count.
func (*MockCluster) UpdateStorageReadBytes ¶
func (mc *MockCluster) UpdateStorageReadBytes(storeID uint64, BytesRead uint64)
UpdateStorageReadBytes updates store read bytes.
func (*MockCluster) UpdateStorageWrittenBytes ¶
func (mc *MockCluster) UpdateStorageWrittenBytes(storeID uint64, BytesWritten uint64)
UpdateStorageWrittenBytes updates store written bytes.
func (*MockCluster) UpdateStoreLeaderSize ¶
func (mc *MockCluster) UpdateStoreLeaderSize(storeID uint64, size int64)
UpdateStoreLeaderSize updates store leader size.
func (*MockCluster) UpdateStoreLeaderWeight ¶
func (mc *MockCluster) UpdateStoreLeaderWeight(storeID uint64, weight float64)
UpdateStoreLeaderWeight updates store leader weight.
func (*MockCluster) UpdateStoreRegionSize ¶
func (mc *MockCluster) UpdateStoreRegionSize(storeID uint64, size int64)
UpdateStoreRegionSize updates store region size.
func (*MockCluster) UpdateStoreRegionWeight ¶
func (mc *MockCluster) UpdateStoreRegionWeight(storeID uint64, weight float64)
UpdateStoreRegionWeight updates store region weight.
func (*MockCluster) UpdateStoreStatus ¶
func (mc *MockCluster) UpdateStoreStatus(id uint64)
UpdateStoreStatus updates store status.
type MockSchedulerOptions ¶
type MockSchedulerOptions struct { RegionScheduleLimit uint64 LeaderScheduleLimit uint64 ReplicaScheduleLimit uint64 MergeScheduleLimit uint64 HotRegionScheduleLimit uint64 MaxSnapshotCount uint64 MaxPendingPeerCount uint64 MaxMergeRegionSize uint64 MaxMergeRegionKeys uint64 SplitMergeInterval time.Duration MaxStoreDownTime time.Duration MaxReplicas int LocationLabels []string HotRegionCacheHitsThreshold int TolerantSizeRatio float64 LowSpaceRatio float64 HighSpaceRatio float64 DisableLearner bool DisableRemoveDownReplica bool DisableReplaceOfflineReplica bool DisableMakeUpReplica bool DisableRemoveExtraReplica bool DisableLocationReplacement bool DisableNamespaceRelocation bool LabelProperties map[string][]*metapb.StoreLabel }
MockSchedulerOptions is a mock of SchedulerOptions which implements Options interface
func NewMockSchedulerOptions ¶
func NewMockSchedulerOptions() *MockSchedulerOptions
NewMockSchedulerOptions creates a mock schedule option.
func (*MockSchedulerOptions) GetHighSpaceRatio ¶
func (mso *MockSchedulerOptions) GetHighSpaceRatio() float64
GetHighSpaceRatio mock method
func (*MockSchedulerOptions) GetHotRegionCacheHitsThreshold ¶
func (mso *MockSchedulerOptions) GetHotRegionCacheHitsThreshold() int
GetHotRegionCacheHitsThreshold mock method
func (*MockSchedulerOptions) GetHotRegionScheduleLimit ¶
func (mso *MockSchedulerOptions) GetHotRegionScheduleLimit(name string) uint64
GetHotRegionScheduleLimit mock method
func (*MockSchedulerOptions) GetLeaderScheduleLimit ¶
func (mso *MockSchedulerOptions) GetLeaderScheduleLimit(name string) uint64
GetLeaderScheduleLimit mock method
func (*MockSchedulerOptions) GetLocationLabels ¶
func (mso *MockSchedulerOptions) GetLocationLabels() []string
GetLocationLabels mock method
func (*MockSchedulerOptions) GetLowSpaceRatio ¶
func (mso *MockSchedulerOptions) GetLowSpaceRatio() float64
GetLowSpaceRatio mock method
func (*MockSchedulerOptions) GetMaxMergeRegionKeys ¶
func (mso *MockSchedulerOptions) GetMaxMergeRegionKeys() uint64
GetMaxMergeRegionKeys mock method
func (*MockSchedulerOptions) GetMaxMergeRegionSize ¶
func (mso *MockSchedulerOptions) GetMaxMergeRegionSize() uint64
GetMaxMergeRegionSize mock method
func (*MockSchedulerOptions) GetMaxPendingPeerCount ¶
func (mso *MockSchedulerOptions) GetMaxPendingPeerCount() uint64
GetMaxPendingPeerCount mock method
func (*MockSchedulerOptions) GetMaxReplicas ¶
func (mso *MockSchedulerOptions) GetMaxReplicas(name string) int
GetMaxReplicas mock method
func (*MockSchedulerOptions) GetMaxSnapshotCount ¶
func (mso *MockSchedulerOptions) GetMaxSnapshotCount() uint64
GetMaxSnapshotCount mock method
func (*MockSchedulerOptions) GetMaxStoreDownTime ¶
func (mso *MockSchedulerOptions) GetMaxStoreDownTime() time.Duration
GetMaxStoreDownTime mock method
func (*MockSchedulerOptions) GetMergeScheduleLimit ¶
func (mso *MockSchedulerOptions) GetMergeScheduleLimit(name string) uint64
GetMergeScheduleLimit mock method
func (*MockSchedulerOptions) GetRegionScheduleLimit ¶
func (mso *MockSchedulerOptions) GetRegionScheduleLimit(name string) uint64
GetRegionScheduleLimit mock method
func (*MockSchedulerOptions) GetReplicaScheduleLimit ¶
func (mso *MockSchedulerOptions) GetReplicaScheduleLimit(name string) uint64
GetReplicaScheduleLimit mock method
func (*MockSchedulerOptions) GetSplitMergeInterval ¶
func (mso *MockSchedulerOptions) GetSplitMergeInterval() time.Duration
GetSplitMergeInterval mock method
func (*MockSchedulerOptions) GetTolerantSizeRatio ¶
func (mso *MockSchedulerOptions) GetTolerantSizeRatio() float64
GetTolerantSizeRatio mock method
func (*MockSchedulerOptions) IsLocationReplacementEnabled ¶
func (mso *MockSchedulerOptions) IsLocationReplacementEnabled() bool
IsLocationReplacementEnabled mock method.
func (*MockSchedulerOptions) IsMakeUpReplicaEnabled ¶
func (mso *MockSchedulerOptions) IsMakeUpReplicaEnabled() bool
IsMakeUpReplicaEnabled mock method.
func (*MockSchedulerOptions) IsNamespaceRelocationEnabled ¶
func (mso *MockSchedulerOptions) IsNamespaceRelocationEnabled() bool
IsNamespaceRelocationEnabled mock method.
func (*MockSchedulerOptions) IsRaftLearnerEnabled ¶
func (mso *MockSchedulerOptions) IsRaftLearnerEnabled() bool
IsRaftLearnerEnabled mock method
func (*MockSchedulerOptions) IsRemoveDownReplicaEnabled ¶
func (mso *MockSchedulerOptions) IsRemoveDownReplicaEnabled() bool
IsRemoveDownReplicaEnabled mock method.
func (*MockSchedulerOptions) IsRemoveExtraReplicaEnabled ¶
func (mso *MockSchedulerOptions) IsRemoveExtraReplicaEnabled() bool
IsRemoveExtraReplicaEnabled mock method.
func (*MockSchedulerOptions) IsReplaceOfflineReplicaEnabled ¶
func (mso *MockSchedulerOptions) IsReplaceOfflineReplicaEnabled() bool
IsReplaceOfflineReplicaEnabled mock method.
func (*MockSchedulerOptions) SetMaxReplicas ¶
func (mso *MockSchedulerOptions) SetMaxReplicas(replicas int)
SetMaxReplicas mock method
type NamespaceChecker ¶
type NamespaceChecker struct {
// contains filtered or unexported fields
}
NamespaceChecker ensures region to go to the right place.
func NewNamespaceChecker ¶
func NewNamespaceChecker(cluster Cluster, classifier namespace.Classifier) *NamespaceChecker
NewNamespaceChecker creates a namespace checker.
func (*NamespaceChecker) Check ¶
func (n *NamespaceChecker) Check(region *core.RegionInfo) *Operator
Check verifies a region's namespace, creating an Operator if need.
func (*NamespaceChecker) SelectBestPeerToRelocate ¶
func (n *NamespaceChecker) SelectBestPeerToRelocate(region *core.RegionInfo, targets []*core.StoreInfo) *metapb.Peer
SelectBestPeerToRelocate return a new peer that to be used to move a region
func (*NamespaceChecker) SelectBestStoreToRelocate ¶
func (n *NamespaceChecker) SelectBestStoreToRelocate(region *core.RegionInfo, targets []*core.StoreInfo) uint64
SelectBestStoreToRelocate randomly returns the store to relocate
type NamespaceOptions ¶
type NamespaceOptions interface { GetLeaderScheduleLimit(name string) uint64 GetRegionScheduleLimit(name string) uint64 GetReplicaScheduleLimit(name string) uint64 GetMergeScheduleLimit(name string) uint64 GetMaxReplicas(name string) int }
NamespaceOptions for namespace cluster.
type OpInfluence ¶
type OpInfluence struct {
// contains filtered or unexported fields
}
OpInfluence records the influence of the cluster.
func NewOpInfluence ¶
func NewOpInfluence(operators []*Operator, cluster Cluster) OpInfluence
NewOpInfluence creates a OpInfluence.
func (OpInfluence) GetRegionsInfluence ¶
func (m OpInfluence) GetRegionsInfluence() map[uint64]*Operator
GetRegionsInfluence gets regionInfluence of specific region.
func (OpInfluence) GetStoreInfluence ¶
func (m OpInfluence) GetStoreInfluence(id uint64) *StoreInfluence
GetStoreInfluence get storeInfluence of specific store.
type Operator ¶
type Operator struct {
// contains filtered or unexported fields
}
Operator contains execution steps generated by scheduler.
func CreateMovePeerOperator ¶
func CreateMovePeerOperator(desc string, cluster Cluster, region *core.RegionInfo, kind OperatorKind, oldStore, newStore uint64, peerID uint64) (*Operator, error)
CreateMovePeerOperator creates an Operator that replaces an old peer with a new peer.
func CreateRemovePeerOperator ¶
func CreateRemovePeerOperator(desc string, cluster Cluster, kind OperatorKind, region *core.RegionInfo, storeID uint64) (*Operator, error)
CreateRemovePeerOperator creates an Operator that removes a peer from region.
func NewOperator ¶
func NewOperator(desc string, regionID uint64, regionEpoch *metapb.RegionEpoch, kind OperatorKind, steps ...OperatorStep) *Operator
NewOperator creates a new operator.
func (*Operator) AttachKind ¶
func (o *Operator) AttachKind(kind OperatorKind)
AttachKind attaches an operator kind for the operator.
func (*Operator) Check ¶
func (o *Operator) Check(region *core.RegionInfo) OperatorStep
Check checks if current step is finished, returns next step to take action. It's safe to be called by multiple goroutine concurrently.
func (*Operator) Desc ¶
Desc returns the operator's short description.
func (*Operator) ElapsedTime ¶
ElapsedTime returns duration since it was created.
func (*Operator) GetPriorityLevel ¶
func (o *Operator) GetPriorityLevel() core.PriorityLevel
GetPriorityLevel get the priority level
func (*Operator) History ¶
func (o *Operator) History() []OperatorHistory
History transfers the operator's steps to operator histories.
func (*Operator) Influence ¶
func (o *Operator) Influence(opInfluence OpInfluence, region *core.RegionInfo)
Influence calculates the store difference which unfinished operator steps make
func (*Operator) IsFinish ¶
IsFinish checks if all steps are finished.
func (*Operator) IsTimeout ¶
IsTimeout checks the operator's create time and determines if it is timeout.
func (*Operator) MarshalJSON ¶
MarshalJSON serialize custom types to JSON
func (*Operator) RegionEpoch ¶
func (o *Operator) RegionEpoch() *metapb.RegionEpoch
RegionEpoch returns the region's epoch that is attched to the operator.
func (*Operator) RegionID ¶
RegionID returns the region that operator is targeted.
func (*Operator) SetDesc ¶
SetDesc sets the description for the operator.
func (*Operator) SetPriorityLevel ¶
func (o *Operator) SetPriorityLevel(level core.PriorityLevel)
SetPriorityLevel set the priority level for operator
type OperatorHistory ¶
type OperatorHistory struct { FinishTime time.Time From, To uint64 Kind core.ResourceKind }
OperatorHistory is used to log and visualize completed operators.
type OperatorKind ¶
type OperatorKind uint32
OperatorKind is a bit field to identify operator types.
const ( OpLeader OperatorKind = 1 << iota // Include leader transfer. OpRegion // Include peer movement. OpAdmin // Initiated by admin. OpHotRegion // Initiated by hot region scheduler. OpAdjacent // Initiated by adjacent region scheduler. OpReplica // Initiated by replica checkers. OpBalance // Initiated by balancers. OpMerge // Initiated by merge checkers or merge schedulers. OpRange // Initiated by range scheduler. )
Flags for operators.
func ParseOperatorKind ¶
func ParseOperatorKind(str string) (OperatorKind, error)
ParseOperatorKind converts string (flag name list concat by ',') to OperatorKind.
func (OperatorKind) String ¶
func (k OperatorKind) String() string
type OperatorStep ¶
type OperatorStep interface { fmt.Stringer IsFinish(region *core.RegionInfo) bool Influence(opInfluence OpInfluence, region *core.RegionInfo) }
OperatorStep describes the basic scheduling steps that can not be subdivided.
func CreateAddPeerSteps ¶
func CreateAddPeerSteps(newStore uint64, peerID uint64, cluster Cluster) []OperatorStep
CreateAddPeerSteps creates an OperatorStep list that add a new Peer.
type Options ¶
type Options interface { GetLeaderScheduleLimit() uint64 GetRegionScheduleLimit() uint64 GetReplicaScheduleLimit() uint64 GetMergeScheduleLimit() uint64 GetHotRegionScheduleLimit() uint64 GetMaxSnapshotCount() uint64 GetMaxPendingPeerCount() uint64 GetMaxStoreDownTime() time.Duration GetMaxMergeRegionSize() uint64 GetMaxMergeRegionKeys() uint64 GetSplitMergeInterval() time.Duration GetMaxReplicas() int GetLocationLabels() []string GetHotRegionCacheHitsThreshold() int GetTolerantSizeRatio() float64 GetLowSpaceRatio() float64 GetHighSpaceRatio() float64 IsRaftLearnerEnabled() bool IsRemoveDownReplicaEnabled() bool IsReplaceOfflineReplicaEnabled() bool IsMakeUpReplicaEnabled() bool IsRemoveExtraReplicaEnabled() bool IsLocationReplacementEnabled() bool IsNamespaceRelocationEnabled() bool CheckLabelProperty(typ string, labels []*metapb.StoreLabel) bool }
Options for schedulers.
type PromoteLearner ¶
type PromoteLearner struct {
ToStore, PeerID uint64
}
PromoteLearner is an OperatorStep that promotes a region learner peer to normal voter.
func (PromoteLearner) Influence ¶
func (pl PromoteLearner) Influence(opInfluence OpInfluence, region *core.RegionInfo)
Influence calculates the store difference that current step make
func (PromoteLearner) IsFinish ¶
func (pl PromoteLearner) IsFinish(region *core.RegionInfo) bool
IsFinish checks if current step is finished.
func (PromoteLearner) String ¶
func (pl PromoteLearner) String() string
type RandomSelector ¶
type RandomSelector struct {
// contains filtered or unexported fields
}
RandomSelector selects source/target store randomly.
func NewRandomSelector ¶
func NewRandomSelector(filters []Filter) *RandomSelector
NewRandomSelector creates a RandomSelector instance.
func (*RandomSelector) SelectSource ¶
SelectSource randomly selects a source store from those can pass all filters.
type RangeCluster ¶
type RangeCluster struct { Cluster // contains filtered or unexported fields }
RangeCluster isolates the cluster by range.
func GenRangeCluster ¶
func GenRangeCluster(cluster Cluster, startKey, endKey []byte) *RangeCluster
GenRangeCluster gets a range cluster by specifying start key and end key.
func (*RangeCluster) GetAverageRegionSize ¶
func (r *RangeCluster) GetAverageRegionSize() int64
GetAverageRegionSize returns the average region approximate size.
func (*RangeCluster) GetFollowerStores ¶
func (r *RangeCluster) GetFollowerStores(region *core.RegionInfo) []*core.StoreInfo
GetFollowerStores returns all stores that contains the region's follower peer.
func (*RangeCluster) GetLeaderStore ¶
func (r *RangeCluster) GetLeaderStore(region *core.RegionInfo) *core.StoreInfo
GetLeaderStore returns all stores that contains the region's leader peer.
func (*RangeCluster) GetRegionStores ¶
func (r *RangeCluster) GetRegionStores(region *core.RegionInfo) []*core.StoreInfo
GetRegionStores returns all stores that contains the region's peer.
func (*RangeCluster) GetStore ¶
func (r *RangeCluster) GetStore(id uint64) *core.StoreInfo
GetStore searches for a store by ID.
func (*RangeCluster) GetStores ¶
func (r *RangeCluster) GetStores() []*core.StoreInfo
GetStores returns all Stores in the cluster.
func (*RangeCluster) GetTolerantSizeRatio ¶
func (r *RangeCluster) GetTolerantSizeRatio() float64
GetTolerantSizeRatio gets the tolerant size ratio.
func (*RangeCluster) RandFollowerRegion ¶
func (r *RangeCluster) RandFollowerRegion(storeID uint64, opts ...core.RegionOption) *core.RegionInfo
RandFollowerRegion returns a random region that has a follower on the store.
func (*RangeCluster) RandLeaderRegion ¶
func (r *RangeCluster) RandLeaderRegion(storeID uint64, opts ...core.RegionOption) *core.RegionInfo
RandLeaderRegion returns a random region that has leader on the store.
func (*RangeCluster) SetTolerantSizeRatio ¶
func (r *RangeCluster) SetTolerantSizeRatio(ratio float64)
SetTolerantSizeRatio sets the tolerant size ratio.
type RegionScatterer ¶
type RegionScatterer struct {
// contains filtered or unexported fields
}
RegionScatterer scatters regions.
func NewRegionScatterer ¶
func NewRegionScatterer(cluster Cluster, classifier namespace.Classifier) *RegionScatterer
NewRegionScatterer creates a region scatterer.
func (*RegionScatterer) Scatter ¶
func (r *RegionScatterer) Scatter(region *core.RegionInfo) (*Operator, error)
Scatter relocates the region.
type RemovePeer ¶
type RemovePeer struct {
FromStore uint64
}
RemovePeer is an OperatorStep that removes a region peer.
func (RemovePeer) Influence ¶
func (rp RemovePeer) Influence(opInfluence OpInfluence, region *core.RegionInfo)
Influence calculates the store difference that current step make
func (RemovePeer) IsFinish ¶
func (rp RemovePeer) IsFinish(region *core.RegionInfo) bool
IsFinish checks if current step is finished.
func (RemovePeer) String ¶
func (rp RemovePeer) String() string
type ReplicaChecker ¶
type ReplicaChecker struct {
// contains filtered or unexported fields
}
ReplicaChecker ensures region has the best replicas.
func NewReplicaChecker ¶
func NewReplicaChecker(cluster Cluster, classifier namespace.Classifier) *ReplicaChecker
NewReplicaChecker creates a replica checker.
func (*ReplicaChecker) Check ¶
func (r *ReplicaChecker) Check(region *core.RegionInfo) *Operator
Check verifies a region's replicas, creating an Operator if need.
func (*ReplicaChecker) SelectBestReplacementStore ¶
func (r *ReplicaChecker) SelectBestReplacementStore(region *core.RegionInfo, oldPeer *metapb.Peer, filters ...Filter) (uint64, float64)
SelectBestReplacementStore returns a store id that to be used to replace the old peer and distinct score.
type ReplicaSelector ¶
type ReplicaSelector struct {
// contains filtered or unexported fields
}
ReplicaSelector selects source/target store candidates based on their distinct scores based on a region's peer stores.
func NewReplicaSelector ¶
func NewReplicaSelector(regionStores []*core.StoreInfo, labels []string, filters ...Filter) *ReplicaSelector
NewReplicaSelector creates a ReplicaSelector instance.
func (*ReplicaSelector) SelectSource ¶
SelectSource selects the store that can pass all filters and has the minimal distinct score.
type Scheduler ¶
type Scheduler interface { GetName() string // GetType should in accordance with the name passing to schedule.RegisterScheduler() GetType() string GetMinInterval() time.Duration GetNextInterval(interval time.Duration) time.Duration Prepare(cluster Cluster) error Cleanup(cluster Cluster) Schedule(cluster Cluster, opInfluence OpInfluence) []*Operator IsScheduleAllowed(cluster Cluster) bool }
Scheduler is an interface to schedule resources.
type SplitRegion ¶
type SplitRegion struct {
StartKey, EndKey []byte
Policy pdpb.CheckPolicy
}
SplitRegion is an OperatorStep that splits a region.
func (SplitRegion) Influence ¶
func (sr SplitRegion) Influence(opInfluence OpInfluence, region *core.RegionInfo)
Influence calculates the store difference that current step make.
func (SplitRegion) IsFinish ¶
func (sr SplitRegion) IsFinish(region *core.RegionInfo) bool
IsFinish checks if current step is finished.
func (SplitRegion) String ¶
func (sr SplitRegion) String() string
type StoreInfluence ¶
type StoreInfluence struct { RegionSize int64 RegionCount int64 LeaderSize int64 LeaderCount int64 }
StoreInfluence records influences that pending operators will make.
func (StoreInfluence) ResourceSize ¶
func (s StoreInfluence) ResourceSize(kind core.ResourceKind) int64
ResourceSize returns delta size of leader/region by influence.
type StoreStateFilter ¶
type StoreStateFilter struct { // Set true if the schedule involves any transfer leader operation. TransferLeader bool // Set true if the schedule involves any move region operation. MoveRegion bool }
StoreStateFilter is used to determine whether a store can be selected as the source or target of the schedule based on the store's state.
func (StoreStateFilter) FilterSource ¶
func (f StoreStateFilter) FilterSource(opt Options, store *core.StoreInfo) bool
FilterSource returns true when the store cannot be selected as the schedule source.
func (StoreStateFilter) FilterTarget ¶
func (f StoreStateFilter) FilterTarget(opt Options, store *core.StoreInfo) bool
FilterTarget returns true when the store cannot be selected as the schedule target.
func (StoreStateFilter) Type ¶
func (f StoreStateFilter) Type() string
Type returns the type of the Filter.
type TransferLeader ¶
type TransferLeader struct {
FromStore, ToStore uint64
}
TransferLeader is an OperatorStep that transfers a region's leader.
func (TransferLeader) Influence ¶
func (tl TransferLeader) Influence(opInfluence OpInfluence, region *core.RegionInfo)
Influence calculates the store difference that current step make
func (TransferLeader) IsFinish ¶
func (tl TransferLeader) IsFinish(region *core.RegionInfo) bool
IsFinish checks if current step is finished.
func (TransferLeader) String ¶
func (tl TransferLeader) String() string