Documentation ¶
Overview ¶
Package topo is the module responsible for interacting with the topology service. It uses one Conn connection to the global topo service (with possibly another one to a read-only version of the global topo service), and one to each cell topo service.
It contains the plug-in interfaces Conn, Factory and Version that topo implementations will use. We support Zookeeper, etcd, consul as real topo servers, and in-memory, tee as test and utility topo servers. Implementations are in sub-directories here.
In tests, we do not mock this package. Instead, we just use a memorytopo.
We also support copying data across topo servers (using helpers/copy.go and the topo2topo cmd binary), and writing to two topo servers at the same time (using helpers/tee.go). This is to facilitate migrations between topo servers.
There are two test sub-packages associated with this code:
- test/ contains a test suite that is run against all of our implementations. It just performs a bunch of common topo server activities (create, list, delete various objects, ...). If a topo implementation passes all these tests, it most likely will work as expected in a real deployment.
- topotests/ contains tests that use a memorytopo to test the code in this package.
Index ¶
- Constants
- Variables
- func CheckKeyspaceLocked(ctx context.Context, keyspace string) error
- func CheckShardLocked(ctx context.Context, keyspace, shard string) error
- func DeleteTabletReplicationData(ctx context.Context, ts *Server, tablet *topodatapb.Tablet) error
- func DirEntriesSortByName(entries []DirEntry)
- func DirEntriesToStringArray(entries []DirEntry) []string
- func FixShardReplication(ctx context.Context, ts *Server, logger logutil.Logger, ...) error
- func GetAliasByCell(ctx context.Context, ts *Server, cell string) string
- func GetExternalVitessClusterDir() string
- func GetExternalVitessClusterPath(clusterName string) string
- func InCellList(cell string, cells []string) bool
- func IsErrType(err error, code ErrorCode) bool
- func IsInServingGraph(tt topodatapb.TabletType) bool
- func IsReplicaType(tt topodatapb.TabletType) bool
- func IsRunningQueryService(tt topodatapb.TabletType) bool
- func IsRunningUpdateStream(tt topodatapb.TabletType) bool
- func IsShardUsingRangeBasedSharding(shard string) bool
- func IsSubjectToLameduck(tt topodatapb.TabletType) bool
- func IsTrivialTypeChange(oldTabletType, newTabletType topodatapb.TabletType) bool
- func NewError(code ErrorCode, node string) error
- func NewTablet(uid uint32, cell, host string) *topodatapb.Tablet
- func OrderAndCheckPartitions(cell string, srvKeyspace *topodatapb.SrvKeyspace) error
- func ParseServingTabletType(param string) (topodatapb.TabletType, error)
- func RegisterFactory(name string, factory Factory)
- func RemoveShardReplicationRecord(ctx context.Context, ts *Server, cell, keyspace, shard string, ...) error
- func ShardIsServing(srvKeyspace *topodatapb.SrvKeyspace, shard *topodatapb.Shard) bool
- func TabletEquality(left, right *topodatapb.Tablet) bool
- func UpdateShardReplicationRecord(ctx context.Context, ts *Server, keyspace, shard string, ...) error
- func UpdateTabletReplicationData(ctx context.Context, ts *Server, tablet *topodatapb.Tablet) error
- func Validate(ctx context.Context, ts *Server, tabletAlias *topodatapb.TabletAlias) error
- func ValidateShardName(shard string) (string, *topodatapb.KeyRange, error)
- type CancelFunc
- type Conn
- type DirEntry
- type DirEntryType
- type Error
- type ErrorCode
- type ExternalVitessClusterInfo
- type Factory
- type KeyspaceInfo
- func (ki *KeyspaceInfo) CheckServedFromMigration(tabletType topodatapb.TabletType, cells []string, keyspace string, remove bool) error
- func (ki *KeyspaceInfo) ComputeCellServedFrom(cell string) []*topodatapb.SrvKeyspace_ServedFrom
- func (ki *KeyspaceInfo) GetServedFrom(tabletType topodatapb.TabletType) *topodatapb.Keyspace_ServedFrom
- func (ki *KeyspaceInfo) KeyspaceName() string
- func (ki *KeyspaceInfo) UpdateServedFromMap(tabletType topodatapb.TabletType, cells []string, keyspace string, remove bool, ...) error
- type KeyspaceShard
- type Lock
- type LockDescriptor
- type MasterParticipation
- type Server
- func (ts *Server) AddSrvKeyspacePartitions(ctx context.Context, keyspace string, shards []*ShardInfo, ...) (err error)
- func (ts *Server) Close()
- func (ts *Server) ConnForCell(ctx context.Context, cell string) (Conn, error)
- func (ts *Server) CreateCellInfo(ctx context.Context, cell string, ci *topodatapb.CellInfo) error
- func (ts *Server) CreateCellsAlias(ctx context.Context, alias string, cellsAlias *topodatapb.CellsAlias) error
- func (ts *Server) CreateExternalVitessCluster(ctx context.Context, clusterName string, ...) error
- func (ts *Server) CreateKeyspace(ctx context.Context, keyspace string, value *topodatapb.Keyspace) error
- func (ts *Server) CreateShard(ctx context.Context, keyspace, shard string) (err error)
- func (ts *Server) CreateTablet(ctx context.Context, tablet *topodatapb.Tablet) error
- func (ts *Server) CreateWorkflow(ctx context.Context, w *workflowpb.Workflow) (*WorkflowInfo, error)
- func (ts *Server) DeleteCellInfo(ctx context.Context, cell string, force bool) error
- func (ts *Server) DeleteCellsAlias(ctx context.Context, alias string) error
- func (ts *Server) DeleteExternalVitessCluster(ctx context.Context, clusterName string) error
- func (ts *Server) DeleteKeyspace(ctx context.Context, keyspace string) error
- func (ts *Server) DeleteKeyspaceReplication(ctx context.Context, cell, keyspace string) error
- func (ts *Server) DeleteMetadata(ctx context.Context, key string) error
- func (ts *Server) DeleteShard(ctx context.Context, keyspace, shard string) error
- func (ts *Server) DeleteShardReplication(ctx context.Context, cell, keyspace, shard string) error
- func (ts *Server) DeleteSrvKeyspace(ctx context.Context, cell, keyspace string) error
- func (ts *Server) DeleteSrvKeyspacePartitions(ctx context.Context, keyspace string, shards []*ShardInfo, ...) (err error)
- func (ts *Server) DeleteSrvVSchema(ctx context.Context, cell string) error
- func (ts *Server) DeleteTablet(ctx context.Context, tabletAlias *topodatapb.TabletAlias) error
- func (ts *Server) DeleteVSchema(ctx context.Context, keyspace string) error
- func (ts *Server) DeleteWorkflow(ctx context.Context, wi *WorkflowInfo) error
- func (ts *Server) EnsureVSchema(ctx context.Context, keyspace string) error
- func (ts *Server) ExpandCells(ctx context.Context, cells string) ([]string, error)
- func (ts *Server) FindAllShardsInKeyspace(ctx context.Context, keyspace string) (map[string]*ShardInfo, error)
- func (ts *Server) FindAllTabletAliasesInShard(ctx context.Context, keyspace, shard string) ([]*topodatapb.TabletAlias, error)
- func (ts *Server) FindAllTabletAliasesInShardByCell(ctx context.Context, keyspace, shard string, cells []string) ([]*topodatapb.TabletAlias, error)
- func (ts *Server) GetCellInfo(ctx context.Context, cell string, strongRead bool) (*topodatapb.CellInfo, error)
- func (ts *Server) GetCellInfoNames(ctx context.Context) ([]string, error)
- func (ts *Server) GetCellsAlias(ctx context.Context, name string, strongRead bool) (*topodatapb.CellsAlias, error)
- func (ts *Server) GetCellsAliases(ctx context.Context, strongRead bool) (ret map[string]*topodatapb.CellsAlias, err error)
- func (ts *Server) GetExternalVitessCluster(ctx context.Context, clusterName string) (*ExternalVitessClusterInfo, error)
- func (ts *Server) GetExternalVitessClusters(ctx context.Context) ([]string, error)
- func (ts *Server) GetKeyspace(ctx context.Context, keyspace string) (*KeyspaceInfo, error)
- func (ts *Server) GetKeyspaces(ctx context.Context) ([]string, error)
- func (ts *Server) GetKnownCells(ctx context.Context) ([]string, error)
- func (ts *Server) GetMetadata(ctx context.Context, keyFilter string) (map[string]string, error)
- func (ts *Server) GetOnlyShard(ctx context.Context, keyspace string) (*ShardInfo, error)
- func (ts *Server) GetOrCreateShard(ctx context.Context, keyspace, shard string) (si *ShardInfo, err error)
- func (ts *Server) GetRoutingRules(ctx context.Context) (*vschemapb.RoutingRules, error)
- func (ts *Server) GetServingShards(ctx context.Context, keyspace string) ([]*ShardInfo, error)
- func (ts *Server) GetShard(ctx context.Context, keyspace, shard string) (*ShardInfo, error)
- func (ts *Server) GetShardNames(ctx context.Context, keyspace string) ([]string, error)
- func (ts *Server) GetShardReplication(ctx context.Context, cell, keyspace, shard string) (*ShardReplicationInfo, error)
- func (ts *Server) GetShardServingCells(ctx context.Context, si *ShardInfo) (servingCells []string, err error)
- func (ts *Server) GetShardServingTypes(ctx context.Context, si *ShardInfo) (servingTypes []topodatapb.TabletType, err error)
- func (ts *Server) GetSrvKeyspace(ctx context.Context, cell, keyspace string) (*topodatapb.SrvKeyspace, error)
- func (ts *Server) GetSrvKeyspaceAllCells(ctx context.Context, keyspace string) ([]*topodatapb.SrvKeyspace, error)
- func (ts *Server) GetSrvKeyspaceNames(ctx context.Context, cell string) ([]string, error)
- func (ts *Server) GetSrvVSchema(ctx context.Context, cell string) (*vschemapb.SrvVSchema, error)
- func (ts *Server) GetTablet(ctx context.Context, alias *topodatapb.TabletAlias) (*TabletInfo, error)
- func (ts *Server) GetTabletMap(ctx context.Context, tabletAliases []*topodatapb.TabletAlias) (map[string]*TabletInfo, error)
- func (ts *Server) GetTabletMapForShard(ctx context.Context, keyspace, shard string) (map[string]*TabletInfo, error)
- func (ts *Server) GetTabletMapForShardByCell(ctx context.Context, keyspace, shard string, cells []string) (map[string]*TabletInfo, error)
- func (ts *Server) GetTabletsByCell(ctx context.Context, cell string) ([]*topodatapb.TabletAlias, error)
- func (ts *Server) GetVSchema(ctx context.Context, keyspace string) (*vschemapb.Keyspace, error)
- func (ts *Server) GetWorkflow(ctx context.Context, uuid string) (*WorkflowInfo, error)
- func (ts *Server) GetWorkflowNames(ctx context.Context) ([]string, error)
- func (ts *Server) LockKeyspace(ctx context.Context, keyspace, action string) (context.Context, func(*error), error)
- func (ts *Server) LockShard(ctx context.Context, keyspace, shard, action string) (context.Context, func(*error), error)
- func (ts *Server) MigrateServedType(ctx context.Context, keyspace string, shardsToAdd, shardsToRemove []*ShardInfo, ...) (err error)
- func (ts *Server) OpenExternalVitessClusterServer(ctx context.Context, clusterName string) (*Server, error)
- func (ts *Server) RebuildSrvVSchema(ctx context.Context, cells []string) error
- func (ts *Server) ResolveKeyspaceWildcard(ctx context.Context, param string) ([]string, error)
- func (ts *Server) ResolveShardWildcard(ctx context.Context, param string) ([]KeyspaceShard, error)
- func (ts *Server) ResolveWildcards(ctx context.Context, cell string, paths []string) ([]string, error)
- func (ts *Server) SaveRoutingRules(ctx context.Context, routingRules *vschemapb.RoutingRules) error
- func (ts *Server) SaveVSchema(ctx context.Context, keyspace string, vschema *vschemapb.Keyspace) error
- func (ts *Server) SaveWorkflow(ctx context.Context, wi *WorkflowInfo) error
- func (ts *Server) UpdateCellInfoFields(ctx context.Context, cell string, update func(*topodatapb.CellInfo) error) error
- func (ts *Server) UpdateCellsAlias(ctx context.Context, alias string, update func(*topodatapb.CellsAlias) error) error
- func (ts *Server) UpdateDisableQueryService(ctx context.Context, keyspace string, shards []*ShardInfo, ...) (err error)
- func (ts *Server) UpdateExternalVitessCluster(ctx context.Context, vc *ExternalVitessClusterInfo) error
- func (ts *Server) UpdateKeyspace(ctx context.Context, ki *KeyspaceInfo) error
- func (ts *Server) UpdateShardFields(ctx context.Context, keyspace, shard string, update func(*ShardInfo) error) (*ShardInfo, error)
- func (ts *Server) UpdateShardReplicationFields(ctx context.Context, cell, keyspace, shard string, ...) error
- func (ts *Server) UpdateSrvKeyspace(ctx context.Context, cell, keyspace string, ...) error
- func (ts *Server) UpdateSrvVSchema(ctx context.Context, cell string, srvVSchema *vschemapb.SrvVSchema) error
- func (ts *Server) UpdateTablet(ctx context.Context, ti *TabletInfo) error
- func (ts *Server) UpdateTabletFields(ctx context.Context, alias *topodatapb.TabletAlias, ...) (*topodatapb.Tablet, error)
- func (ts *Server) UpsertMetadata(ctx context.Context, key string, val string) error
- func (ts *Server) ValidateSrvKeyspace(ctx context.Context, keyspace, cells string) error
- func (ts *Server) WatchShard(ctx context.Context, keyspace, shard string) (*WatchShardData, <-chan *WatchShardData, CancelFunc)
- func (ts *Server) WatchSrvKeyspace(ctx context.Context, cell, keyspace string) (*WatchSrvKeyspaceData, <-chan *WatchSrvKeyspaceData, CancelFunc)
- func (ts *Server) WatchSrvVSchema(ctx context.Context, cell string) (*WatchSrvVSchemaData, <-chan *WatchSrvVSchemaData, CancelFunc)
- type ShardInfo
- func (si *ShardInfo) GetPrimaryTermStartTime() time.Time
- func (si *ShardInfo) GetTabletControl(tabletType topodatapb.TabletType) *topodatapb.Shard_TabletControl
- func (si *ShardInfo) HasPrimary() bool
- func (si *ShardInfo) Keyspace() string
- func (si *ShardInfo) SetPrimaryTermStartTime(t time.Time)
- func (si *ShardInfo) ShardName() string
- func (si *ShardInfo) UpdateSourceDeniedTables(ctx context.Context, tabletType topodatapb.TabletType, cells []string, ...) error
- func (si *ShardInfo) Version() Version
- type ShardReplicationInfo
- type StatsConn
- func (st *StatsConn) Close()
- func (st *StatsConn) Create(ctx context.Context, filePath string, contents []byte) (Version, error)
- func (st *StatsConn) Delete(ctx context.Context, filePath string, version Version) error
- func (st *StatsConn) Get(ctx context.Context, filePath string) ([]byte, Version, error)
- func (st *StatsConn) ListDir(ctx context.Context, dirPath string, full bool) ([]DirEntry, error)
- func (st *StatsConn) Lock(ctx context.Context, dirPath, contents string) (LockDescriptor, error)
- func (st *StatsConn) NewMasterParticipation(name, id string) (MasterParticipation, error)
- func (st *StatsConn) Update(ctx context.Context, filePath string, contents []byte, version Version) (Version, error)
- func (st *StatsConn) Watch(ctx context.Context, filePath string) (current *WatchData, changes <-chan *WatchData, cancel CancelFunc)
- type TabletInfo
- func (ti *TabletInfo) Addr() string
- func (ti *TabletInfo) AliasString() string
- func (ti *TabletInfo) DbName() string
- func (ti *TabletInfo) GetPrimaryTermStartTime() time.Time
- func (ti *TabletInfo) IsInServingGraph() bool
- func (ti *TabletInfo) IsReplicaType() bool
- func (ti *TabletInfo) MysqlAddr() string
- func (ti *TabletInfo) String() string
- func (ti *TabletInfo) Version() Version
- type Version
- type WatchData
- type WatchShardData
- type WatchSrvKeyspaceData
- type WatchSrvVSchemaData
- type WorkflowInfo
Constants ¶
const ( NodeExists = ErrorCode(iota) NoNode NodeNotEmpty Timeout Interrupted BadVersion PartialResult NoUpdateNeeded NoImplementation )
The following is the list of error codes.
const ( // GlobalCell is the name of the global cell. It is special // as it contains the global topology, and references the other cells. GlobalCell = "global" // GlobalReadOnlyCell is the name of the global read-only cell // connection cell name. GlobalReadOnlyCell = "global-read-only" )
const ( CellInfoFile = "CellInfo" CellsAliasFile = "CellsAlias" KeyspaceFile = "Keyspace" ShardFile = "Shard" VSchemaFile = "VSchema" ShardReplicationFile = "ShardReplication" TabletFile = "Tablet" SrvVSchemaFile = "SrvVSchema" SrvKeyspaceFile = "SrvKeyspace" RoutingRulesFile = "RoutingRules" ExternalClustersFile = "ExternalClusters" )
Filenames for all object types.
const ( CellsPath = "cells" CellsAliasesPath = "cells_aliases" KeyspacesPath = "keyspaces" ShardsPath = "shards" TabletsPath = "tablets" MetadataPath = "metadata" ExternalClusterMySQL = "mysql" ExternalClusterVitess = "vitess" )
Path for all object types.
Variables ¶
var ( // RemoteOperationTimeout is used for operations where we have to // call out to another process. // Used for RPC calls (including topo server calls) RemoteOperationTimeout = flag.Duration("remote_operation_timeout", 30*time.Second, "time to wait for a remote operation") )
Functions ¶
func CheckKeyspaceLocked ¶
CheckKeyspaceLocked can be called on a context to make sure we have the lock for a given keyspace.
func CheckShardLocked ¶
CheckShardLocked can be called on a context to make sure we have the lock for a given shard.
func DeleteTabletReplicationData ¶
DeleteTabletReplicationData deletes replication data.
func DirEntriesSortByName ¶
func DirEntriesSortByName(entries []DirEntry)
DirEntriesSortByName sorts a slice of DirEntry objects by Name.
func DirEntriesToStringArray ¶
DirEntriesToStringArray is a helper method to extract the names from an []DirEntry
func FixShardReplication ¶
func FixShardReplication(ctx context.Context, ts *Server, logger logutil.Logger, cell, keyspace, shard string) error
FixShardReplication will fix the first problem it encounters within a ShardReplication object.
func GetAliasByCell ¶
GetAliasByCell returns the alias group this `cell` belongs to, if there's none, it returns the `cell` as alias.
func GetExternalVitessClusterDir ¶ added in v0.10.0
func GetExternalVitessClusterDir() string
GetExternalVitessClusterDir returns node path containing external vitess clusters
func GetExternalVitessClusterPath ¶ added in v0.10.0
GetExternalVitessClusterPath returns node path containing external clusters
func InCellList ¶
InCellList returns true if the cell list is empty, or if the passed cell is in the cell list.
func IsInServingGraph ¶
func IsInServingGraph(tt topodatapb.TabletType) bool
IsInServingGraph returns if a tablet appears in the serving graph
func IsReplicaType ¶
func IsReplicaType(tt topodatapb.TabletType) bool
IsReplicaType returns if this type should be connected to a primary db and actively replicating? PRIMARY is not obviously (only support one level replication graph) BACKUP, RESTORE, DRAINED may or may not be, but we don't know for sure
func IsRunningQueryService ¶
func IsRunningQueryService(tt topodatapb.TabletType) bool
IsRunningQueryService returns if a tablet is running the query service
func IsRunningUpdateStream ¶
func IsRunningUpdateStream(tt topodatapb.TabletType) bool
IsRunningUpdateStream returns if a tablet is running the update stream RPC service.
func IsShardUsingRangeBasedSharding ¶
IsShardUsingRangeBasedSharding returns true if the shard name implies it is using range based sharding.
func IsSubjectToLameduck ¶
func IsSubjectToLameduck(tt topodatapb.TabletType) bool
IsSubjectToLameduck returns if a tablet is subject to being lameduck. Lameduck is a transition period where we are still allowed to serve, but we tell the clients we are going away soon. Typically, a vttablet will still serve, but broadcast a non-serving state through its health check. then vtgate will catch that non-serving state, and stop sending queries.
Primaries are not subject to lameduck, as we usually want to transition them as fast as possible.
Replica and rdonly will use lameduck when going from healthy to unhealthy (either because health check fails, or they're shutting down).
Other types are probably not serving user visible traffic, so they need to transition as fast as possible too.
func IsTrivialTypeChange ¶
func IsTrivialTypeChange(oldTabletType, newTabletType topodatapb.TabletType) bool
IsTrivialTypeChange returns if this db type be trivially reassigned without changes to the replication graph
func NewTablet ¶
func NewTablet(uid uint32, cell, host string) *topodatapb.Tablet
NewTablet create a new Tablet record with the given id, cell, and hostname.
func OrderAndCheckPartitions ¶
func OrderAndCheckPartitions(cell string, srvKeyspace *topodatapb.SrvKeyspace) error
OrderAndCheckPartitions will re-order the partition list, and check it's correct.
func ParseServingTabletType ¶ added in v0.10.0
func ParseServingTabletType(param string) (topodatapb.TabletType, error)
ParseServingTabletType parses the tablet type into the enum, and makes sure that the enum is of serving type (PRIMARY, REPLICA, RDONLY/BATCH).
Note: This function more closely belongs in topoproto, but that would create a circular import between packages topo and topoproto.
func RegisterFactory ¶
RegisterFactory registers a Factory for an implementation for a Server. If an implementation with that name already exists, it log.Fatals out. Call this in the 'init' function in your topology implementation module.
func RemoveShardReplicationRecord ¶
func RemoveShardReplicationRecord(ctx context.Context, ts *Server, cell, keyspace, shard string, tabletAlias *topodatapb.TabletAlias) error
RemoveShardReplicationRecord is a low level function to remove an entry from the ShardReplication object.
func ShardIsServing ¶
func ShardIsServing(srvKeyspace *topodatapb.SrvKeyspace, shard *topodatapb.Shard) bool
ShardIsServing returns true if this shard is found in any of the partitions in the srvKeyspace
func TabletEquality ¶
func TabletEquality(left, right *topodatapb.Tablet) bool
TabletEquality returns true iff two Tablet are representing the same tablet process: same uid/cell, running on the same host / ports.
func UpdateShardReplicationRecord ¶
func UpdateShardReplicationRecord(ctx context.Context, ts *Server, keyspace, shard string, tabletAlias *topodatapb.TabletAlias) error
UpdateShardReplicationRecord is a low level function to add / update an entry to the ShardReplication object.
func UpdateTabletReplicationData ¶
UpdateTabletReplicationData creates or updates the replication graph data for a tablet
func Validate ¶
func Validate(ctx context.Context, ts *Server, tabletAlias *topodatapb.TabletAlias) error
Validate makes sure a tablet is represented correctly in the topology server.
func ValidateShardName ¶
func ValidateShardName(shard string) (string, *topodatapb.KeyRange, error)
ValidateShardName takes a shard name and sanitizes it, and also returns the KeyRange.
Types ¶
type Conn ¶
type Conn interface { // ListDir returns the entries in a directory. The returned // list should be sorted by entry.Name. // If there are no files under the provided path, returns ErrNoNode. // dirPath is a path relative to the root directory of the cell. // If full is set, we want all the fields in DirEntry to be filled in. // If full is not set, only Name will be used. This is intended for // implementations where getting more than the names is more expensive, // as in most cases only the names are needed. ListDir(ctx context.Context, dirPath string, full bool) ([]DirEntry, error) // Create creates the initial version of a file. // Returns ErrNodeExists if the file exists. // filePath is a path relative to the root directory of the cell. Create(ctx context.Context, filePath string, contents []byte) (Version, error) // Update updates the file with the provided filename with the // new content. // If version is nil, it is an unconditional update // (which is then the same as a Create is the file doesn't exist). // filePath is a path relative to the root directory of the cell. // It returns the new Version of the file after update. // Returns ErrBadVersion if the provided version is not current. Update(ctx context.Context, filePath string, contents []byte, version Version) (Version, error) // Get returns the content and version of a file. // filePath is a path relative to the root directory of the cell. // Can return ErrNoNode if the file doesn't exist. Get(ctx context.Context, filePath string) ([]byte, Version, error) // Delete deletes the provided file. // If version is nil, it is an unconditional delete. // If the last entry of a directory is deleted, using ListDir // on its parent directory should not return the directory. // For instance, when deleting /keyspaces/aaa/Keyspace, and if // there is no other file in /keyspaces/aaa, then aaa should not // appear any more when listing /keyspaces. // filePath is a path relative to the root directory of the cell. // // Delete will never be called on a directory. // Returns ErrNodeExists if the file doesn't exist. // Returns ErrBadVersion if the provided version is not current. Delete(ctx context.Context, filePath string, version Version) error // Lock takes a lock on the given directory. // It does not prevent any modification to any file in the topology. // It just prevents two concurrent processes (wherever they are) // to run concurrently. It is used for instance to make sure only // one reparent operation is running on a Shard at a given time. // dirPath is the directory associated with a resource, for instance // a Keyspace or a Shard. It is not a file location. // (this means the implementation can for instance create a // file in this directory to materialize the lock). // contents describes the lock holder and purpose, but has no other // meaning, so it can be used as a lock file contents, for instance. // Returns ErrNoNode if the directory doesn't exist (meaning // there is no existing file under that directory). // Returns ErrTimeout if ctx expires. // Returns ErrInterrupted if ctx is canceled. Lock(ctx context.Context, dirPath, contents string) (LockDescriptor, error) // Watch starts watching a file in the provided cell. It // returns the current value, a 'changes' channel to read the // changes from, and a 'cancel' function to call to stop the // watch. If the initial read fails, or the file doesn't // exist, current.Err is set, and 'changes'/'cancel' are nil. // Otherwise current.Err is nil, and current.Contents / // current.Version are accurate. The provided context is only // used to setup the current watch, and not after Watch() // returns. // // To stop the watch, just call the returned 'cancel' function. // This will eventually result in a final WatchData result with Err = // ErrInterrupted. It should be safe to call the 'cancel' function // multiple times, or after the Watch already errored out. // // The 'changes' channel may return a record with Err != nil. // In that case, the channel will also be closed right after // that record. In any case, 'changes' has to be drained of // all events, even when 'stop' is closed. // // Note the 'changes' channel can return twice the same // Version/Contents (for instance, if the watch is interrupted // and restarted within the Conn implementation). // Similarly, the 'changes' channel may skip versions / changes // (that is, if value goes [A, B, C, D, E, F], the watch may only // receive [A, B, F]). This should only happen for rapidly // changing values though. Usually, the initial value will come // back right away. And a stable value (that hasn't changed for // a while) should be seen shortly. // // The Watch call is not guaranteed to return exactly up to // date data right away. For instance, if a file is created // and saved, and then a watch is set on that file, it may // return ErrNoNode (as the underlying configuration service // may use asynchronous caches that are not up to date // yet). The only guarantee is that the watch data will // eventually converge. Vitess doesn't explicitly depend on the data // being correct quickly, as long as it eventually gets there. // // filePath is a path relative to the root directory of the cell. Watch(ctx context.Context, filePath string) (current *WatchData, changes <-chan *WatchData, cancel CancelFunc) // NewMasterParticipation creates a MasterParticipation // object, used to become the Master in an election for the // provided group name. Id is the name of the local process, // passing in the hostname:port of the current process as id // is the common usage. Id must be unique for each process // calling this, for a given name. Calling this function does // not make the current process a candidate for the election. NewMasterParticipation(name, id string) (MasterParticipation, error) // Close closes the connection to the server. Close() }
Conn defines the interface that must be implemented by topology plug-ins to be used with Vitess.
Zookeeper is a good example of an implementation, as defined in go/vt/topo/zk2topo.
This API is very generic, and key/value store oriented. We use regular paths for object names, and we can list all immediate children of a path. All paths sent through this API are relative paths, from the root directory of the cell.
The Conn objects are created by the Factory implementations.
type DirEntry ¶
type DirEntry struct { // Name is the name of the entry. // Always filled in. Name string // Type is the DirEntryType of the entry. // Only filled in if full is true. Type DirEntryType // Ephemeral is set if the directory / file only contains // data that was not set by the file API, like lock files // or primary-election related files. // Only filled in if full is true. Ephemeral bool }
DirEntry is an entry in a directory, as returned by ListDir.
type DirEntryType ¶
type DirEntryType int
DirEntryType is the type of an entry in a directory.
const ( // TypeDirectory describes a directory. TypeDirectory DirEntryType = iota // TypeFile describes a file. TypeFile )
type Error ¶
type Error struct {
// contains filtered or unexported fields
}
Error represents a topo error.
type ExternalVitessClusterInfo ¶ added in v0.10.0
type ExternalVitessClusterInfo struct { ClusterName string *topodatapb.ExternalVitessCluster // contains filtered or unexported fields }
ExternalVitessClusterInfo is a meta struct that contains metadata to give the data more context and convenience. This is the main way we interact with a vitess cluster stored in the topo.
type Factory ¶
type Factory interface { // HasGlobalReadOnlyCell returns true if the global cell // has read-only replicas of the topology data. The global topology // is usually more expensive to read from / write to, as it is // replicated over many cells. Some topology services provide // more efficient way to read the data, like Observer servers // for Zookeeper. If this returns true, we will maintain // two connections for the global topology: the 'global' cell // for consistent reads and writes, and the 'global-read-only' // cell for reads only. HasGlobalReadOnlyCell(serverAddr, root string) bool // Create creates a topo.Conn object. Create(cell, serverAddr, root string) (Conn, error) }
Factory is a factory interface to create Conn objects. Topo implementations will provide an implementation for this.
type KeyspaceInfo ¶
type KeyspaceInfo struct { *topodatapb.Keyspace // contains filtered or unexported fields }
KeyspaceInfo is a meta struct that contains metadata to give the data more context and convenience. This is the main way we interact with a keyspace.
func (*KeyspaceInfo) CheckServedFromMigration ¶
func (ki *KeyspaceInfo) CheckServedFromMigration(tabletType topodatapb.TabletType, cells []string, keyspace string, remove bool) error
CheckServedFromMigration makes sure a requested migration is safe
func (*KeyspaceInfo) ComputeCellServedFrom ¶
func (ki *KeyspaceInfo) ComputeCellServedFrom(cell string) []*topodatapb.SrvKeyspace_ServedFrom
ComputeCellServedFrom returns the ServedFrom list for a cell
func (*KeyspaceInfo) GetServedFrom ¶
func (ki *KeyspaceInfo) GetServedFrom(tabletType topodatapb.TabletType) *topodatapb.Keyspace_ServedFrom
GetServedFrom returns a Keyspace_ServedFrom record if it exists.
func (*KeyspaceInfo) KeyspaceName ¶
func (ki *KeyspaceInfo) KeyspaceName() string
KeyspaceName returns the keyspace name
func (*KeyspaceInfo) UpdateServedFromMap ¶
func (ki *KeyspaceInfo) UpdateServedFromMap(tabletType topodatapb.TabletType, cells []string, keyspace string, remove bool, allCells []string) error
UpdateServedFromMap handles ServedFromMap. It can add or remove records, cells, ...
type KeyspaceShard ¶
KeyspaceShard is a type used by ResolveShardWildcard
type Lock ¶
type Lock struct { // Action and the following fields are set at construction time. Action string HostName string UserName string Time string // Status is the current status of the Lock. Status string }
Lock describes a long-running lock on a keyspace or a shard. It needs to be public as we JSON-serialize it.
type LockDescriptor ¶
type LockDescriptor interface { // Check returns an error if the lock was lost. // Some topology implementations use a keep-alive mechanism, and // sometimes it fails. The users of the lock are responsible for // checking on it when convenient. Check(ctx context.Context) error // Unlock releases the lock. Unlock(ctx context.Context) error }
LockDescriptor is an interface that describes a lock. It will be returned by Lock().
type MasterParticipation ¶
type MasterParticipation interface { // WaitForMastership makes the current process a candidate // for election, and waits until this process is the primary. // After we become the primary, we may lose primaryship. In that case, // the returned context will be canceled. If Stop was called, // WaitForMastership will return nil, ErrInterrupted. WaitForMastership() (context.Context, error) // Stop is called when we don't want to participate in the // primary election any more. Typically, that is when the // hosting process is terminating. We will relinquish // primaryship at that point, if we had it. Stop should // not return until everything has been done. // The MasterParticipation object should be discarded // after Stop has been called. Any call to WaitForMastership // after Stop() will return nil, ErrInterrupted. // If WaitForMastership() was running, it will return // nil, ErrInterrupted as soon as possible. Stop() // GetCurrentMasterID returns the current primary id. // This may not work after Stop has been called. GetCurrentMasterID(ctx context.Context) (string, error) }
MasterParticipation is the object returned by NewMasterParticipation. Sample usage:
mp := server.NewMasterParticipation("vtctld", "hostname:8080") job := NewJob()
go func() { for { ctx, err := mp.WaitForMastership() switch err { case nil: job.RunUntilContextDone(ctx) case topo.ErrInterrupted: return default: log.Errorf("Got error while waiting for primary, will retry in 5s: %v", err) time.Sleep(5 * time.Second) } } }()
http.HandleFunc("/", func(w http.ResponseWriter, r *http.Request) { if job.Running() { job.WriteStatus(w, r) } else { http.Redirect(w, r, mp.GetCurrentMasterID(context.Background()), http.StatusFound) } })
servenv.OnTermSync(func() { mp.Stop() })
type Server ¶
type Server struct {
// contains filtered or unexported fields
}
Server is the main topo.Server object. We support two ways of creating one:
- From an implementation, server address, and root path. This uses a plugin mechanism, and we have implementations for etcd, zookeeper and consul.
- Specific implementations may have higher level creation methods (in which case they may provide a more complex Factory). We support memorytopo (for tests and processes that only need an in-memory server), and tee (a helper implementation to transition between one server implementation and another).
func NewWithFactory ¶
NewWithFactory creates a new Server based on the given Factory. It also opens the global cell connection.
func Open ¶
func Open() *Server
Open returns a Server using the command line parameter flags for implementation, address and root. It log.Exits out if an error occurs.
func OpenServer ¶
OpenServer returns a Server using the provided implementation, address and root for the global server.
func (*Server) AddSrvKeyspacePartitions ¶
func (ts *Server) AddSrvKeyspacePartitions(ctx context.Context, keyspace string, shards []*ShardInfo, tabletType topodatapb.TabletType, cells []string) (err error)
AddSrvKeyspacePartitions adds partitions to srvKeyspace
func (*Server) Close ¶
func (ts *Server) Close()
Close will close all connections to underlying topo Server. It will nil all member variables, so any further access will panic.
func (*Server) ConnForCell ¶
ConnForCell returns a Conn object for the given cell. It caches Conn objects from previously requested cells.
func (*Server) CreateCellInfo ¶
CreateCellInfo creates a new CellInfo with the provided content.
func (*Server) CreateCellsAlias ¶
func (ts *Server) CreateCellsAlias(ctx context.Context, alias string, cellsAlias *topodatapb.CellsAlias) error
CreateCellsAlias creates a new CellInfo with the provided content.
func (*Server) CreateExternalVitessCluster ¶ added in v0.10.0
func (ts *Server) CreateExternalVitessCluster(ctx context.Context, clusterName string, value *topodatapb.ExternalVitessCluster) error
CreateExternalVitessCluster creates a topo record for the passed vitess cluster
func (*Server) CreateKeyspace ¶
func (ts *Server) CreateKeyspace(ctx context.Context, keyspace string, value *topodatapb.Keyspace) error
CreateKeyspace wraps the underlying Conn.Create and dispatches the event.
func (*Server) CreateShard ¶
CreateShard creates a new shard and tries to fill in the right information. This will lock the Keyspace, as we may be looking at other shard servedTypes. Using GetOrCreateShard is probably a better idea for most use cases.
func (*Server) CreateTablet ¶
CreateTablet creates a new tablet and all associated paths for the replication graph.
func (*Server) CreateWorkflow ¶
func (ts *Server) CreateWorkflow(ctx context.Context, w *workflowpb.Workflow) (*WorkflowInfo, error)
CreateWorkflow creates the given workflow, and returns the initial WorkflowInfo.
func (*Server) DeleteCellInfo ¶
DeleteCellInfo deletes the specified CellInfo. We first try to make sure no Shard record points to the cell, but we'll continue regardless if 'force' is true.
func (*Server) DeleteCellsAlias ¶
DeleteCellsAlias deletes the specified CellsAlias
func (*Server) DeleteExternalVitessCluster ¶ added in v0.10.0
DeleteExternalVitessCluster deletes the topo record for the named vitess cluster
func (*Server) DeleteKeyspace ¶
DeleteKeyspace wraps the underlying Conn.Delete and dispatches the event.
func (*Server) DeleteKeyspaceReplication ¶
DeleteKeyspaceReplication deletes all the ShardReplication objects for a cell/keyspace.
func (*Server) DeleteMetadata ¶
DeleteMetadata deletes the key in the metadata
func (*Server) DeleteShard ¶
DeleteShard wraps the underlying conn.Delete and dispatches the event.
func (*Server) DeleteShardReplication ¶
DeleteShardReplication deletes a ShardReplication object.
func (*Server) DeleteSrvKeyspace ¶
DeleteSrvKeyspace deletes a SrvKeyspace.
func (*Server) DeleteSrvKeyspacePartitions ¶
func (ts *Server) DeleteSrvKeyspacePartitions(ctx context.Context, keyspace string, shards []*ShardInfo, tabletType topodatapb.TabletType, cells []string) (err error)
DeleteSrvKeyspacePartitions deletes shards from srvKeyspace partitions
func (*Server) DeleteSrvVSchema ¶
DeleteSrvVSchema deletes the SrvVSchema file for a cell.
func (*Server) DeleteTablet ¶
func (ts *Server) DeleteTablet(ctx context.Context, tabletAlias *topodatapb.TabletAlias) error
DeleteTablet wraps the underlying conn.Delete and dispatches the event.
func (*Server) DeleteVSchema ¶
DeleteVSchema delete the keyspace if it exists
func (*Server) DeleteWorkflow ¶
func (ts *Server) DeleteWorkflow(ctx context.Context, wi *WorkflowInfo) error
DeleteWorkflow deletes the specified workflow. After this, the WorkflowInfo object should not be used any more.
func (*Server) EnsureVSchema ¶
EnsureVSchema makes sure that a vschema is present for this keyspace or creates a blank one if it is missing
func (*Server) ExpandCells ¶ added in v0.10.0
ExpandCells takes a comma-separated list of cells and returns an array of cell names Aliases are expanded and an empty string returns all cells
func (*Server) FindAllShardsInKeyspace ¶
func (ts *Server) FindAllShardsInKeyspace(ctx context.Context, keyspace string) (map[string]*ShardInfo, error)
FindAllShardsInKeyspace reads and returns all the existing shards in a keyspace. It doesn't take any lock.
func (*Server) FindAllTabletAliasesInShard ¶
func (ts *Server) FindAllTabletAliasesInShard(ctx context.Context, keyspace, shard string) ([]*topodatapb.TabletAlias, error)
FindAllTabletAliasesInShard uses the replication graph to find all the tablet aliases in the given shard.
It can return ErrPartialResult if some cells were not fetched, in which case the result only contains the cells that were fetched.
The tablet aliases are sorted by cell, then by UID.
func (*Server) FindAllTabletAliasesInShardByCell ¶
func (ts *Server) FindAllTabletAliasesInShardByCell(ctx context.Context, keyspace, shard string, cells []string) ([]*topodatapb.TabletAlias, error)
FindAllTabletAliasesInShardByCell uses the replication graph to find all the tablet aliases in the given shard.
It can return ErrPartialResult if some cells were not fetched, in which case the result only contains the cells that were fetched.
The tablet aliases are sorted by cell, then by UID.
func (*Server) GetCellInfo ¶
func (ts *Server) GetCellInfo(ctx context.Context, cell string, strongRead bool) (*topodatapb.CellInfo, error)
GetCellInfo reads a CellInfo from the global Conn.
func (*Server) GetCellInfoNames ¶
GetCellInfoNames returns the names of the existing cells. They are sorted by name.
func (*Server) GetCellsAlias ¶
func (ts *Server) GetCellsAlias(ctx context.Context, name string, strongRead bool) (*topodatapb.CellsAlias, error)
GetCellsAlias returns the CellsAlias that matches the given name.
func (*Server) GetCellsAliases ¶
func (ts *Server) GetCellsAliases(ctx context.Context, strongRead bool) (ret map[string]*topodatapb.CellsAlias, err error)
GetCellsAliases returns the names of the existing cells. They are sorted by name.
func (*Server) GetExternalVitessCluster ¶ added in v0.10.0
func (ts *Server) GetExternalVitessCluster(ctx context.Context, clusterName string) (*ExternalVitessClusterInfo, error)
GetExternalVitessCluster returns a topo record for the named vitess cluster
func (*Server) GetExternalVitessClusters ¶ added in v0.10.0
GetExternalVitessClusters returns the list of external vitess clusters in the topology.
func (*Server) GetKeyspace ¶
GetKeyspace reads the given keyspace and returns it
func (*Server) GetKeyspaces ¶
GetKeyspaces returns the list of keyspaces in the topology.
func (*Server) GetKnownCells ¶
GetKnownCells returns the list of known cells. For now, it just lists the 'cells' directory in the global topology server. TODO(alainjobart) once the cell map is migrated to this generic package, we can do better than this.
func (*Server) GetMetadata ¶
GetMetadata retrieves all metadata value that matches the given key regular expression. If empty all values are returned.
func (*Server) GetOnlyShard ¶
GetOnlyShard returns the single ShardInfo of an unsharded keyspace.
func (*Server) GetOrCreateShard ¶
func (ts *Server) GetOrCreateShard(ctx context.Context, keyspace, shard string) (si *ShardInfo, err error)
GetOrCreateShard will return the shard object, or create one if it doesn't already exist. Note the shard creation is protected by a keyspace Lock.
func (*Server) GetRoutingRules ¶
GetRoutingRules fetches the routing rules from the topo.
func (*Server) GetServingShards ¶
GetServingShards returns all shards where the primary is serving.
func (*Server) GetShard ¶
GetShard is a high level function to read shard data. It generates trace spans.
func (*Server) GetShardNames ¶
GetShardNames returns the list of shards in a keyspace.
func (*Server) GetShardReplication ¶
func (ts *Server) GetShardReplication(ctx context.Context, cell, keyspace, shard string) (*ShardReplicationInfo, error)
GetShardReplication returns the ShardReplicationInfo object.
func (*Server) GetShardServingCells ¶
func (ts *Server) GetShardServingCells(ctx context.Context, si *ShardInfo) (servingCells []string, err error)
GetShardServingCells returns cells where this shard is serving
func (*Server) GetShardServingTypes ¶
func (ts *Server) GetShardServingTypes(ctx context.Context, si *ShardInfo) (servingTypes []topodatapb.TabletType, err error)
GetShardServingTypes returns served types for given shard across all cells
func (*Server) GetSrvKeyspace ¶
func (ts *Server) GetSrvKeyspace(ctx context.Context, cell, keyspace string) (*topodatapb.SrvKeyspace, error)
GetSrvKeyspace returns the SrvKeyspace for a cell/keyspace.
func (*Server) GetSrvKeyspaceAllCells ¶
func (ts *Server) GetSrvKeyspaceAllCells(ctx context.Context, keyspace string) ([]*topodatapb.SrvKeyspace, error)
GetSrvKeyspaceAllCells returns the SrvKeyspace for all cells
func (*Server) GetSrvKeyspaceNames ¶
GetSrvKeyspaceNames returns the SrvKeyspace objects for a cell.
func (*Server) GetSrvVSchema ¶
GetSrvVSchema returns the SrvVSchema for a cell.
func (*Server) GetTablet ¶
func (ts *Server) GetTablet(ctx context.Context, alias *topodatapb.TabletAlias) (*TabletInfo, error)
GetTablet is a high level function to read tablet data. It generates trace spans.
func (*Server) GetTabletMap ¶
func (ts *Server) GetTabletMap(ctx context.Context, tabletAliases []*topodatapb.TabletAlias) (map[string]*TabletInfo, error)
GetTabletMap tries to read all the tablets in the provided list, and returns them all in a map. If error is ErrPartialResult, the results in the dictionary are incomplete, meaning some tablets couldn't be read. The map is indexed by topoproto.TabletAliasString(tablet alias).
func (*Server) GetTabletMapForShard ¶
func (ts *Server) GetTabletMapForShard(ctx context.Context, keyspace, shard string) (map[string]*TabletInfo, error)
GetTabletMapForShard returns the tablets for a shard. It can return ErrPartialResult if it couldn't read all the cells, or all the individual tablets, in which case the map is valid, but partial. The map is indexed by topoproto.TabletAliasString(tablet alias).
func (*Server) GetTabletMapForShardByCell ¶
func (ts *Server) GetTabletMapForShardByCell(ctx context.Context, keyspace, shard string, cells []string) (map[string]*TabletInfo, error)
GetTabletMapForShardByCell returns the tablets for a shard. It can return ErrPartialResult if it couldn't read all the cells, or all the individual tablets, in which case the map is valid, but partial. The map is indexed by topoproto.TabletAliasString(tablet alias).
func (*Server) GetTabletsByCell ¶
func (ts *Server) GetTabletsByCell(ctx context.Context, cell string) ([]*topodatapb.TabletAlias, error)
GetTabletsByCell returns all the tablets in a cell. It returns ErrNode if the cell doesn't exist. It returns (nil, nil) if the cell exists, but there are no tablets.
func (*Server) GetVSchema ¶
GetVSchema fetches the vschema from the topo.
func (*Server) GetWorkflow ¶
GetWorkflow reads a workflow from the global cell.
func (*Server) GetWorkflowNames ¶
GetWorkflowNames returns the names of the existing workflows. They are sorted by uuid.
func (*Server) LockKeyspace ¶
func (ts *Server) LockKeyspace(ctx context.Context, keyspace, action string) (context.Context, func(*error), error)
LockKeyspace will lock the keyspace, and return: - a context with a locksInfo structure for future reference. - an unlock method - an error if anything failed.
We lock a keyspace for the following operations to be guaranteed exclusive operation: * changing a keyspace sharding info fields (is this one necessary?) * changing a keyspace 'ServedFrom' field (is this one necessary?) * resharding operations:
- horizontal resharding: includes changing the shard's 'ServedType', as well as the associated horizontal resharding operations.
- vertical resharding: includes changing the keyspace 'ServedFrom' field, as well as the associated vertical resharding operations.
- 'vtctl SetShardIsPrimaryServing' emergency operations
- 'vtctl SetShardTabletControl' emergency operations
- 'vtctl SourceShardAdd' and 'vtctl SourceShardDelete' emergency operations
* keyspace-wide schema changes
func (*Server) LockShard ¶
func (ts *Server) LockShard(ctx context.Context, keyspace, shard, action string) (context.Context, func(*error), error)
LockShard will lock the shard, and return: - a context with a locksInfo structure for future reference. - an unlock method - an error if anything failed.
We are currently only using this method to lock actions that would impact each-other. Most changes of the Shard object are done by UpdateShardFields, which is not locking the shard object. The current list of actions that lock a shard are: * all Vitess-controlled re-parenting operations:
- InitShardPrimary
- PlannedReparentShard
- EmergencyReparentShard
* operations that we don't want to conflict with re-parenting:
- DeleteTablet when it's the shard's current primary
func (*Server) MigrateServedType ¶
func (ts *Server) MigrateServedType(ctx context.Context, keyspace string, shardsToAdd, shardsToRemove []*ShardInfo, tabletType topodatapb.TabletType, cells []string) (err error)
MigrateServedType removes/adds shards from srvKeyspace when migrating a served type.
func (*Server) OpenExternalVitessClusterServer ¶ added in v0.10.0
func (ts *Server) OpenExternalVitessClusterServer(ctx context.Context, clusterName string) (*Server, error)
OpenExternalVitessClusterServer returns the topo server of the external cluster
func (*Server) RebuildSrvVSchema ¶
RebuildVSchema rebuilds the SrvVSchema for the provided cell list (or all cells if cell list is empty).
func (*Server) ResolveKeyspaceWildcard ¶
ResolveKeyspaceWildcard will resolve keyspace wildcards.
- If the param is not a wildcard, it will just be returned (if the keyspace doesn't exist, it is still returned).
- If the param is a wildcard, it will get all keyspaces and returns the ones which match the wildcard (which may be an empty list).
func (*Server) ResolveShardWildcard ¶
ResolveShardWildcard will resolve shard wildcards. Both keyspace and shard names can use wildcard. Errors talking to the topology server are returned. ErrNoNode is ignored if it's the result of resolving a wildcard. Examples:
- */* returns all keyspace/shard pairs, or empty list if none.
- user/* returns all shards in user keyspace (or error if user keyspace doesn't exist)
- us*/* returns all shards in all keyspaces that start with 'us'. If no such keyspace exists, list is empty (it is not an error).
func (*Server) ResolveWildcards ¶
func (ts *Server) ResolveWildcards(ctx context.Context, cell string, paths []string) ([]string, error)
ResolveWildcards resolves paths like: /keyspaces/*/Keyspace into real existing paths
If you send paths that don't contain any wildcard and don't exist, this function will return an empty array.
func (*Server) SaveRoutingRules ¶
SaveRoutingRules saves the routing rules into the topo.
func (*Server) SaveVSchema ¶
func (ts *Server) SaveVSchema(ctx context.Context, keyspace string, vschema *vschemapb.Keyspace) error
SaveVSchema first validates the VSchema, then saves it. If the VSchema is empty, just remove it.
func (*Server) SaveWorkflow ¶
func (ts *Server) SaveWorkflow(ctx context.Context, wi *WorkflowInfo) error
SaveWorkflow saves the WorkflowInfo object. If the version is not good any more, ErrBadVersion is returned.
func (*Server) UpdateCellInfoFields ¶
func (ts *Server) UpdateCellInfoFields(ctx context.Context, cell string, update func(*topodatapb.CellInfo) error) error
UpdateCellInfoFields is a high level helper method to read a CellInfo object, update its fields, and then write it back. If the write fails due to a version mismatch, it will re-read the record and retry the update. If the update method returns ErrNoUpdateNeeded, nothing is written, and nil is returned.
func (*Server) UpdateCellsAlias ¶
func (ts *Server) UpdateCellsAlias(ctx context.Context, alias string, update func(*topodatapb.CellsAlias) error) error
UpdateCellsAlias updates cells for a given alias
func (*Server) UpdateDisableQueryService ¶
func (ts *Server) UpdateDisableQueryService(ctx context.Context, keyspace string, shards []*ShardInfo, tabletType topodatapb.TabletType, cells []string, disableQueryService bool) (err error)
UpdateDisableQueryService will make sure the disableQueryService is set appropriately in tablet controls in srvKeyspace.
func (*Server) UpdateExternalVitessCluster ¶ added in v0.10.0
func (ts *Server) UpdateExternalVitessCluster(ctx context.Context, vc *ExternalVitessClusterInfo) error
UpdateExternalVitessCluster updates the topo record for the named vitess cluster
func (*Server) UpdateKeyspace ¶
func (ts *Server) UpdateKeyspace(ctx context.Context, ki *KeyspaceInfo) error
UpdateKeyspace updates the keyspace data. It checks the keyspace is locked.
func (*Server) UpdateShardFields ¶
func (ts *Server) UpdateShardFields(ctx context.Context, keyspace, shard string, update func(*ShardInfo) error) (*ShardInfo, error)
UpdateShardFields is a high level helper to read a shard record, call an update function on it, and then write it back. If the write fails due to a version mismatch, it will re-read the record and retry the update. If the update succeeds, it returns the updated ShardInfo. If the update method returns ErrNoUpdateNeeded, nothing is written, and nil,nil is returned.
Note the callback method takes a ShardInfo, so it can get the keyspace and shard from it, or use all the ShardInfo methods.
func (*Server) UpdateShardReplicationFields ¶
func (ts *Server) UpdateShardReplicationFields(ctx context.Context, cell, keyspace, shard string, update func(*topodatapb.ShardReplication) error) error
UpdateShardReplicationFields updates the fields inside a topo.ShardReplication object.
func (*Server) UpdateSrvKeyspace ¶
func (ts *Server) UpdateSrvKeyspace(ctx context.Context, cell, keyspace string, srvKeyspace *topodatapb.SrvKeyspace) error
UpdateSrvKeyspace saves a new SrvKeyspace. It is a blind write.
func (*Server) UpdateSrvVSchema ¶
func (ts *Server) UpdateSrvVSchema(ctx context.Context, cell string, srvVSchema *vschemapb.SrvVSchema) error
UpdateSrvVSchema updates the SrvVSchema file for a cell.
func (*Server) UpdateTablet ¶
func (ts *Server) UpdateTablet(ctx context.Context, ti *TabletInfo) error
UpdateTablet updates the tablet data only - not associated replication paths. It also uses a span, and sends the event.
func (*Server) UpdateTabletFields ¶
func (ts *Server) UpdateTabletFields(ctx context.Context, alias *topodatapb.TabletAlias, update func(*topodatapb.Tablet) error) (*topodatapb.Tablet, error)
UpdateTabletFields is a high level helper to read a tablet record, call an update function on it, and then write it back. If the write fails due to a version mismatch, it will re-read the record and retry the update. If the update succeeds, it returns the updated tablet. If the update method returns ErrNoUpdateNeeded, nothing is written, and nil,nil is returned.
func (*Server) UpsertMetadata ¶
UpsertMetadata sets the key/value in the metadata if it doesn't exist, otherwise it updates the content
func (*Server) ValidateSrvKeyspace ¶ added in v0.10.0
ValidateSrvKeyspace validates that the SrvKeyspace for given keyspace in the provided cells is not corrupted
func (*Server) WatchShard ¶
func (ts *Server) WatchShard(ctx context.Context, keyspace, shard string) (*WatchShardData, <-chan *WatchShardData, CancelFunc)
WatchShard will set a watch on the Shard object. It has the same contract as conn.Watch, but it also unpacks the contents into a Shard object
func (*Server) WatchSrvKeyspace ¶
func (ts *Server) WatchSrvKeyspace(ctx context.Context, cell, keyspace string) (*WatchSrvKeyspaceData, <-chan *WatchSrvKeyspaceData, CancelFunc)
WatchSrvKeyspace will set a watch on the SrvKeyspace object. It has the same contract as Conn.Watch, but it also unpacks the contents into a SrvKeyspace object.
func (*Server) WatchSrvVSchema ¶
func (ts *Server) WatchSrvVSchema(ctx context.Context, cell string) (*WatchSrvVSchemaData, <-chan *WatchSrvVSchemaData, CancelFunc)
WatchSrvVSchema will set a watch on the SrvVSchema object. It has the same contract as Conn.Watch, but it also unpacks the contents into a SrvVSchema object.
type ShardInfo ¶
type ShardInfo struct { *topodatapb.Shard // contains filtered or unexported fields }
ShardInfo is a meta struct that contains metadata to give the data more context and convenience. This is the main way we interact with a shard.
func NewShardInfo ¶
func NewShardInfo(keyspace, shard string, value *topodatapb.Shard, version Version) *ShardInfo
NewShardInfo returns a ShardInfo basing on shard with the keyspace / shard. This function should be only used by Server implementations.
func (*ShardInfo) GetPrimaryTermStartTime ¶ added in v0.12.0
GetPrimaryTermStartTime returns the shard's primary term start time as a Time value.
func (*ShardInfo) GetTabletControl ¶
func (si *ShardInfo) GetTabletControl(tabletType topodatapb.TabletType) *topodatapb.Shard_TabletControl
GetTabletControl returns the Shard_TabletControl for the given tablet type, or nil if it is not in the map.
func (*ShardInfo) HasPrimary ¶ added in v0.12.0
HasPrimary returns true if the Shard has an assigned primary.
func (*ShardInfo) SetPrimaryTermStartTime ¶ added in v0.12.0
SetPrimaryTermStartTime sets the shard's primary term start time as a Time value.
func (*ShardInfo) UpdateSourceDeniedTables ¶ added in v0.12.0
func (si *ShardInfo) UpdateSourceDeniedTables(ctx context.Context, tabletType topodatapb.TabletType, cells []string, remove bool, tables []string) error
UpdateSourceDeniedTables will add or remove the listed tables in the shard record's TabletControl structures. Note we don't support a lot of the corner cases:
- only support one table list per shard. If we encounter a different table list that the provided one, we error out.
- we don't support DisableQueryService at the same time as DeniedTables, because it's not used in the same context (vertical vs horizontal sharding)
This function should be called while holding the keyspace lock.
type ShardReplicationInfo ¶
type ShardReplicationInfo struct { *topodatapb.ShardReplication // contains filtered or unexported fields }
ShardReplicationInfo is the companion structure for ShardReplication.
func NewShardReplicationInfo ¶
func NewShardReplicationInfo(sr *topodatapb.ShardReplication, cell, keyspace, shard string) *ShardReplicationInfo
NewShardReplicationInfo is for topo.Server implementations to create the structure
func (*ShardReplicationInfo) Cell ¶
func (sri *ShardReplicationInfo) Cell() string
Cell returns the cell for a ShardReplicationInfo
func (*ShardReplicationInfo) GetShardReplicationNode ¶
func (sri *ShardReplicationInfo) GetShardReplicationNode(tabletAlias *topodatapb.TabletAlias) (*topodatapb.ShardReplication_Node, error)
GetShardReplicationNode finds a node for a given tablet.
func (*ShardReplicationInfo) Keyspace ¶
func (sri *ShardReplicationInfo) Keyspace() string
Keyspace returns the keyspace for a ShardReplicationInfo
func (*ShardReplicationInfo) Shard ¶
func (sri *ShardReplicationInfo) Shard() string
Shard returns the shard for a ShardReplicationInfo
type StatsConn ¶
type StatsConn struct {
// contains filtered or unexported fields
}
The StatsConn is a wrapper for a Conn that emits stats for every operation
func NewStatsConn ¶
NewStatsConn returns a StatsConn
func (*StatsConn) NewMasterParticipation ¶
func (st *StatsConn) NewMasterParticipation(name, id string) (MasterParticipation, error)
NewMasterParticipation is part of the Conn interface
type TabletInfo ¶
type TabletInfo struct { *topodatapb.Tablet // contains filtered or unexported fields }
TabletInfo is the container for a Tablet, read from the topology server.
func NewTabletInfo ¶
func NewTabletInfo(tablet *topodatapb.Tablet, version Version) *TabletInfo
NewTabletInfo returns a TabletInfo basing on tablet with the version set. This function should be only used by Server implementations.
func (*TabletInfo) AliasString ¶
func (ti *TabletInfo) AliasString() string
AliasString returns the string representation of the tablet alias
func (*TabletInfo) DbName ¶
func (ti *TabletInfo) DbName() string
DbName is usually implied by keyspace. Having the shard information in the database name complicates mysql replication.
func (*TabletInfo) GetPrimaryTermStartTime ¶ added in v0.12.0
func (ti *TabletInfo) GetPrimaryTermStartTime() time.Time
GetPrimaryTermStartTime returns the tablet's primary term start time as a Time value.
func (*TabletInfo) IsInServingGraph ¶
func (ti *TabletInfo) IsInServingGraph() bool
IsInServingGraph returns if this tablet is in the serving graph
func (*TabletInfo) IsReplicaType ¶
func (ti *TabletInfo) IsReplicaType() bool
IsReplicaType returns if this tablet's type is a replica
func (*TabletInfo) MysqlAddr ¶
func (ti *TabletInfo) MysqlAddr() string
MysqlAddr returns hostname:mysql port.
func (*TabletInfo) String ¶
func (ti *TabletInfo) String() string
String returns a string describing the tablet.
func (*TabletInfo) Version ¶
func (ti *TabletInfo) Version() Version
Version returns the version of this tablet from last time it was read or updated.
type Version ¶
type Version interface { // String returns a text representation of the version. String() string }
Version is an interface that describes a file version.
type WatchData ¶
type WatchData struct { // Contents has the bytes that were stored by Create // or Update. Contents []byte // Version contains an opaque representation of the Version // of that file. Version Version // Err is set the same way for both the 'current' value // returned by Watch, or the values read on the 'changes' // channel. It can be: // - nil, then Contents and Version are set. // - ErrNoNode if the file doesn't exist. // - ErrInterrupted if 'cancel' was called. // - any other platform-specific error. Err error }
WatchData is the structure returned by the Watch() API. It can contain: a) an error in Err if the call failed (or if the watch was terminated). b) the current or new version of the data.
type WatchShardData ¶
type WatchShardData struct { Value *topodatapb.Shard Err error }
WatchShardData wraps the data we receive on the watch channel The WatchShard API guarantees exactly one of Value or Err will be set.
type WatchSrvKeyspaceData ¶
type WatchSrvKeyspaceData struct { Value *topodatapb.SrvKeyspace Err error }
WatchSrvKeyspaceData is returned / streamed by WatchSrvKeyspace. The WatchSrvKeyspace API guarantees exactly one of Value or Err will be set.
type WatchSrvVSchemaData ¶
type WatchSrvVSchemaData struct { Value *vschemapb.SrvVSchema Err error }
WatchSrvVSchemaData is returned / streamed by WatchSrvVSchema. The WatchSrvVSchema API guarantees exactly one of Value or Err will be set.
type WorkflowInfo ¶
type WorkflowInfo struct { *workflowpb.Workflow // contains filtered or unexported fields }
WorkflowInfo is a meta struct that contains the version of a Workflow.
Source Files ¶
Directories ¶
Path | Synopsis |
---|---|
Package consultopo implements topo.Server with consul as the backend.
|
Package consultopo implements topo.Server with consul as the backend. |
Package etcd2topo implements topo.Server with etcd as the backend.
|
Package etcd2topo implements topo.Server with etcd as the backend. |
Package events defines the structures used for events relating directly to individual topology elements, like keyspaces, shards, and tablets.
|
Package events defines the structures used for events relating directly to individual topology elements, like keyspaces, shards, and tablets. |
Package helpers contains a few utility classes to handle topo.Server objects, and transitions from one topo implementation to another.
|
Package helpers contains a few utility classes to handle topo.Server objects, and transitions from one topo implementation to another. |
Package k8stopo implements topo.Server with the Kubernetes API as the backend.
|
Package k8stopo implements topo.Server with the Kubernetes API as the backend. |
client/clientset/versioned
This package has the automatically generated clientset.
|
This package has the automatically generated clientset. |
client/clientset/versioned/fake
This package has the automatically generated fake clientset.
|
This package has the automatically generated fake clientset. |
client/clientset/versioned/scheme
This package contains the scheme of the automatically generated clientset.
|
This package contains the scheme of the automatically generated clientset. |
client/clientset/versioned/typed/topo/v1beta1
This package has the automatically generated typed clients.
|
This package has the automatically generated typed clients. |
client/clientset/versioned/typed/topo/v1beta1/fake
Package fake has the automatically generated clients.
|
Package fake has the automatically generated clients. |
Package memorytopo contains an implementation of the topo.Factory / topo.Conn interfaces based on an in-memory tree of data.
|
Package memorytopo contains an implementation of the topo.Factory / topo.Conn interfaces based on an in-memory tree of data. |
Package test contains utilities to test topo.Conn implementations.
|
Package test contains utilities to test topo.Conn implementations. |
Package topoproto contains utility functions to deal with the proto3 structures defined in proto/topodata.
|
Package topoproto contains utility functions to deal with the proto3 structures defined in proto/topodata. |
Package topotests contains all the unit tests for the topo.Server code that is based on topo.Conn.
|
Package topotests contains all the unit tests for the topo.Server code that is based on topo.Conn. |