Documentation ¶
Overview ¶
Package vtgate provides query routing rpc services for vttablets.
Index ¶
- Constants
- Variables
- func NewShardError(in error, target *querypb.Target) error
- func RegisterPluginInitializer(initializer func())
- func SetQueryLogger(logger *streamlog.StreamLogger[*logstats.LogStats])
- type Executor
- func (e *Executor) CloseSession(ctx context.Context, safeSession *SafeSession) error
- func (e *Executor) Commit(ctx context.Context, safeSession *SafeSession) error
- func (e *Executor) Execute(ctx context.Context, method string, safeSession *SafeSession, sql string, ...) (result *sqltypes.Result, err error)
- func (e *Executor) ExecuteLock(ctx context.Context, rs *srvtopo.ResolvedShard, query *querypb.BoundQuery, ...) (*sqltypes.Result, error)
- func (e *Executor) ExecuteMessageStream(ctx context.Context, rss []*srvtopo.ResolvedShard, tableName string, ...) error
- func (e *Executor) ExecuteMultiShard(ctx context.Context, primitive engine.Primitive, rss []*srvtopo.ResolvedShard, ...) (qr *sqltypes.Result, errs []error)
- func (e *Executor) ExecuteVStream(ctx context.Context, rss []*srvtopo.ResolvedShard, filter *binlogdatapb.Filter, ...) error
- func (e *Executor) MessageStream(ctx context.Context, keyspace string, shard string, ...) error
- func (e *Executor) ParseDestinationTarget(targetString string) (string, topodatapb.TabletType, key.Destination, error)
- func (e *Executor) Plans() cache.Cache
- func (e *Executor) Prepare(ctx context.Context, method string, safeSession *SafeSession, sql string, ...) (fld []*querypb.Field, err error)
- func (e *Executor) ReleaseLock(ctx context.Context, session *SafeSession) error
- func (e *Executor) SaveVSchema(vschema *vindexes.VSchema, stats *VSchemaStats)
- func (e *Executor) ServeHTTP(response http.ResponseWriter, request *http.Request)
- func (e *Executor) StreamExecute(ctx context.Context, method string, safeSession *SafeSession, sql string, ...) error
- func (e *Executor) StreamExecuteMulti(ctx context.Context, primitive engine.Primitive, query string, ...) []error
- func (e *Executor) VSchema() *vindexes.VSchema
- func (e *Executor) VSchemaStats() *VSchemaStats
- func (e *Executor) WriteScatterStats(w http.ResponseWriter)
- type RegisterVTGate
- type Resolver
- type SafeSession
- func (session *SafeSession) AddAdvisoryLock(name string)
- func (session *SafeSession) AppendOrUpdate(shardSession *vtgatepb.Session_ShardSession, txMode vtgatepb.TransactionMode) error
- func (session *SafeSession) AutocommitApproval() bool
- func (session *SafeSession) CanAddSavepoint() bool
- func (session *SafeSession) ClearAdvisoryLock()
- func (session *SafeSession) ClearWarnings()
- func (session *SafeSession) EnableLogging()
- func (session *SafeSession) FindAndChangeSessionIfInSingleTxMode(keyspace, shard string, tabletType topodatapb.TabletType, ...) (int64, int64, *topodatapb.TabletAlias, error)
- func (session *SafeSession) GetDDLStrategy() string
- func (session *SafeSession) GetOrCreateOptions() *querypb.ExecuteOptions
- func (session *SafeSession) GetPrepareData(name string) *vtgatepb.PrepareData
- func (session *SafeSession) GetQueryTimeout() int64
- func (session *SafeSession) GetSessionEnableSystemSettings() bool
- func (session *SafeSession) GetSessionUUID() string
- func (session *SafeSession) GetSystemVariables(f func(k string, v string))
- func (session *SafeSession) GetUDV(name string) *querypb.BindVariable
- func (session *SafeSession) HasAdvisoryLock() bool
- func (session *SafeSession) HasSystemVariables() (found bool)
- func (session *SafeSession) InLockSession() bool
- func (session *SafeSession) InReservedConn() bool
- func (session *SafeSession) InTransaction() bool
- func (session *SafeSession) IsRollbackSet() bool
- func (session *SafeSession) MustRollback() bool
- func (session *SafeSession) RecordWarning(warning *querypb.QueryWarning)
- func (session *SafeSession) RemoveAdvisoryLock(name string)
- func (session *SafeSession) RemoveInternalSavepoint()
- func (session *SafeSession) Reset()
- func (session *SafeSession) ResetAll()
- func (session *SafeSession) ResetLock()
- func (session *SafeSession) ResetShard(tabletAlias *topodatapb.TabletAlias) error
- func (session *SafeSession) ResetTx()
- func (session *SafeSession) SavePoints() []string
- func (session *SafeSession) SavepointRollback()
- func (session *SafeSession) SetAutocommittable(flag bool)
- func (session *SafeSession) SetCommitOrder(co vtgatepb.CommitOrder)
- func (session *SafeSession) SetDDLStrategy(strategy string)
- func (session *SafeSession) SetLockSession(lockSession *vtgatepb.Session_ShardSession)
- func (session *SafeSession) SetOptions(options *querypb.ExecuteOptions)
- func (session *SafeSession) SetPreQueries() []string
- func (session *SafeSession) SetQueryTimeout(queryTimeout int64)
- func (session *SafeSession) SetReadAfterWriteGTID(vtgtid string)
- func (session *SafeSession) SetReadAfterWriteTimeout(timeout float64)
- func (session *SafeSession) SetReservedConn(reservedConn bool)
- func (session *SafeSession) SetRollback()
- func (session *SafeSession) SetRollbackCommand()
- func (session *SafeSession) SetSavepoint(name string)
- func (session *SafeSession) SetSavepointState(spNeed bool)
- func (session *SafeSession) SetSessionEnableSystemSettings(allow bool)
- func (session *SafeSession) SetSessionTrackGtids(enable bool)
- func (session *SafeSession) SetSystemVariable(name string, expr string)
- func (session *SafeSession) SetTargetString(target string)
- func (session *SafeSession) SetUserDefinedVariable(key string, value *querypb.BindVariable)
- func (session *SafeSession) StorePrepareData(key string, value *vtgatepb.PrepareData)
- func (session *SafeSession) StoreSavepoint(sql string)
- func (session *SafeSession) TimeZone() *time.Location
- func (session *SafeSession) TriggerLockHeartBeat() bool
- func (session *SafeSession) UpdateLockHeartbeat()
- type ScatterConn
- func (stc *ScatterConn) Close() error
- func (stc *ScatterConn) ExecuteLock(ctx context.Context, rs *srvtopo.ResolvedShard, query *querypb.BoundQuery, ...) (*sqltypes.Result, error)
- func (stc *ScatterConn) ExecuteMultiShard(ctx context.Context, primitive engine.Primitive, rss []*srvtopo.ResolvedShard, ...) (qr *sqltypes.Result, errs []error)
- func (stc *ScatterConn) GetGatewayCacheStatus() TabletCacheStatusList
- func (stc *ScatterConn) GetHealthCheckCacheStatus() discovery.TabletsCacheStatusList
- func (stc *ScatterConn) MessageStream(ctx context.Context, rss []*srvtopo.ResolvedShard, name string, ...) error
- func (stc *ScatterConn) StreamExecuteMulti(ctx context.Context, primitive engine.Primitive, query string, ...) []error
- type SchemaInfo
- type TabletCacheStatus
- type TabletCacheStatusList
- type TabletGateway
- func (gw *TabletGateway) CacheStatus() TabletCacheStatusList
- func (gw *TabletGateway) Close(_ context.Context) error
- func (gw *TabletGateway) DefaultConnCollation() collations.ID
- func (gw *TabletGateway) QueryServiceByAlias(alias *topodatapb.TabletAlias, target *querypb.Target) (queryservice.QueryService, error)
- func (gw *TabletGateway) RegisterStats()
- func (gw *TabletGateway) TabletsCacheStatus() discovery.TabletsCacheStatusList
- func (gw *TabletGateway) WaitForTablets(tabletTypesToWait []topodatapb.TabletType) (err error)
- type TabletStatusAggregator
- type TxConn
- func (txc *TxConn) Begin(ctx context.Context, session *SafeSession, ...) error
- func (txc *TxConn) Commit(ctx context.Context, session *SafeSession) error
- func (txc *TxConn) Release(ctx context.Context, session *SafeSession) error
- func (txc *TxConn) ReleaseAll(ctx context.Context, session *SafeSession) error
- func (txc *TxConn) ReleaseLock(ctx context.Context, session *SafeSession) error
- func (txc *TxConn) Resolve(ctx context.Context, dtid string) error
- func (txc *TxConn) Rollback(ctx context.Context, session *SafeSession) error
- type VSchemaKeyspaceStats
- type VSchemaManager
- func (vm *VSchemaManager) GetCurrentSrvVschema() *vschemapb.SrvVSchema
- func (vm *VSchemaManager) Rebuild()
- func (vm *VSchemaManager) UpdateVSchema(ctx context.Context, ksName string, vschema *vschemapb.SrvVSchema) error
- func (vm *VSchemaManager) VSchemaUpdate(v *vschemapb.SrvVSchema, err error) bool
- type VSchemaOperator
- type VSchemaStats
- type VTGate
- func (vtg *VTGate) CloseSession(ctx context.Context, session *vtgatepb.Session) error
- func (vtg *VTGate) Execute(ctx context.Context, session *vtgatepb.Session, sql string, ...) (newSession *vtgatepb.Session, qr *sqltypes.Result, err error)
- func (vtg *VTGate) ExecuteBatch(ctx context.Context, session *vtgatepb.Session, sqlList []string, ...) (*vtgatepb.Session, []sqltypes.QueryResponse, error)
- func (vtg *VTGate) Gateway() *TabletGateway
- func (vtg *VTGate) GetGatewayCacheStatus() TabletCacheStatusList
- func (vtg *VTGate) HandlePanic(err *error)
- func (vtg *VTGate) IsHealthy() error
- func (vtg *VTGate) Prepare(ctx context.Context, session *vtgatepb.Session, sql string, ...) (newSession *vtgatepb.Session, fld []*querypb.Field, err error)
- func (vtg *VTGate) ResolveTransaction(ctx context.Context, dtid string) error
- func (vtg *VTGate) StreamExecute(ctx context.Context, session *vtgatepb.Session, sql string, ...) (*vtgatepb.Session, error)
- func (vtg *VTGate) VSchemaStats() *VSchemaStats
- func (vtg *VTGate) VStream(ctx context.Context, tabletType topodatapb.TabletType, ...) error
Constants ¶
const ( // ExecutorTemplate is the HTML template to display ExecutorStats. ExecutorTemplate = `` /* 2776-byte string literal not displayed */ )
const MaxBufferingRetries = 3
MaxBufferingRetries is to represent max retries on buffering.
const MaxSkew = int64(2)
MaxSkew is the threshold for a skew to be detected. Since MySQL timestamps are in seconds we account for two round-offs: one for the actual event and another while accounting for the clock skew
const ( // StatusTemplate is the display part to use to show // a TabletCacheStatusList. StatusTemplate = `` /* 780-byte string literal not displayed */ )
const ( // VSchemaTemplate is the HTML template to display VSchemaStats. VSchemaTemplate = `` /* 773-byte string literal not displayed */ )
Variables ¶
var ( // QueryLogHandler is the debug UI path for exposing query logs QueryLogHandler = "/debug/querylog" // QueryLogzHandler is the debug UI path for exposing query logs QueryLogzHandler = "/debug/querylogz" // QueryzHandler is the debug UI path for exposing query plan stats QueryzHandler = "/debug/queryz" // QueryLogger enables streaming logging of queries QueryLogger *streamlog.StreamLogger[*logstats.LogStats] )
var ( // CellsToWatch is the list of cells the healthcheck operates over. If it is empty, only the local cell is watched CellsToWatch string )
var RegisterVTGates []RegisterVTGate
RegisterVTGates stores register funcs for VTGate server.
Functions ¶
func NewShardError ¶
NewShardError returns a new error with the shard info amended.
func RegisterPluginInitializer ¶
func RegisterPluginInitializer(initializer func())
RegisterPluginInitializer lets plugins register themselves to be init'ed at servenv.OnRun-time
func SetQueryLogger ¶ added in v0.16.0
func SetQueryLogger(logger *streamlog.StreamLogger[*logstats.LogStats])
Types ¶
type Executor ¶
type Executor struct {
// contains filtered or unexported fields
}
Executor is the engine that executes queries by utilizing the abilities of the underlying vttablets.
func NewExecutor ¶
func NewExecutor( ctx context.Context, serv srvtopo.Server, cell string, resolver *Resolver, normalize, warnOnShardedOnly bool, streamSize int, cacheCfg *cache.Config, schemaTracker SchemaInfo, noScatter bool, pv plancontext.PlannerVersion, ) *Executor
NewExecutor creates a new Executor.
func (*Executor) CloseSession ¶
func (e *Executor) CloseSession(ctx context.Context, safeSession *SafeSession) error
CloseSession releases the current connection, which rollbacks open transactions and closes reserved connections. It is called then the MySQL servers closes the connection to its client.
func (*Executor) Commit ¶ added in v0.8.0
func (e *Executor) Commit(ctx context.Context, safeSession *SafeSession) error
Commit commits the existing transactions
func (*Executor) Execute ¶
func (e *Executor) Execute(ctx context.Context, method string, safeSession *SafeSession, sql string, bindVars map[string]*querypb.BindVariable) (result *sqltypes.Result, err error)
Execute executes a non-streaming query.
func (*Executor) ExecuteLock ¶ added in v0.8.0
func (e *Executor) ExecuteLock(ctx context.Context, rs *srvtopo.ResolvedShard, query *querypb.BoundQuery, session *SafeSession, lockFuncType sqlparser.LockingFuncType) (*sqltypes.Result, error)
ExecuteLock implements the IExecutor interface
func (*Executor) ExecuteMessageStream ¶ added in v0.11.0
func (e *Executor) ExecuteMessageStream(ctx context.Context, rss []*srvtopo.ResolvedShard, tableName string, callback func(reply *sqltypes.Result) error) error
ExecuteMessageStream implements the IExecutor interface
func (*Executor) ExecuteMultiShard ¶
func (e *Executor) ExecuteMultiShard(ctx context.Context, primitive engine.Primitive, rss []*srvtopo.ResolvedShard, queries []*querypb.BoundQuery, session *SafeSession, autocommit bool, ignoreMaxMemoryRows bool) (qr *sqltypes.Result, errs []error)
ExecuteMultiShard implements the IExecutor interface
func (*Executor) ExecuteVStream ¶ added in v0.11.0
func (e *Executor) ExecuteVStream(ctx context.Context, rss []*srvtopo.ResolvedShard, filter *binlogdatapb.Filter, gtid string, callback func(evs []*binlogdatapb.VEvent) error) error
ExecuteVStream implements the IExecutor interface
func (*Executor) MessageStream ¶
func (e *Executor) MessageStream(ctx context.Context, keyspace string, shard string, keyRange *topodatapb.KeyRange, name string, callback func(*sqltypes.Result) error) error
MessageStream is part of the vtgate service API. This is a V2 level API that's sent to the Resolver.
func (*Executor) ParseDestinationTarget ¶
func (e *Executor) ParseDestinationTarget(targetString string) (string, topodatapb.TabletType, key.Destination, error)
ParseDestinationTarget parses destination target string and sets default keyspace if possible.
func (*Executor) Prepare ¶
func (e *Executor) Prepare(ctx context.Context, method string, safeSession *SafeSession, sql string, bindVars map[string]*querypb.BindVariable) (fld []*querypb.Field, err error)
Prepare executes a prepare statements.
func (*Executor) ReleaseLock ¶ added in v0.14.0
func (e *Executor) ReleaseLock(ctx context.Context, session *SafeSession) error
ReleaseLock implements the IExecutor interface
func (*Executor) SaveVSchema ¶
func (e *Executor) SaveVSchema(vschema *vindexes.VSchema, stats *VSchemaStats)
SaveVSchema updates the vschema and stats
func (*Executor) ServeHTTP ¶
func (e *Executor) ServeHTTP(response http.ResponseWriter, request *http.Request)
ServeHTTP shows the current plans in the query cache.
func (*Executor) StreamExecute ¶
func (e *Executor) StreamExecute( ctx context.Context, method string, safeSession *SafeSession, sql string, bindVars map[string]*querypb.BindVariable, callback func(*sqltypes.Result) error, ) error
StreamExecute executes a streaming query.
func (*Executor) StreamExecuteMulti ¶
func (e *Executor) StreamExecuteMulti(ctx context.Context, primitive engine.Primitive, query string, rss []*srvtopo.ResolvedShard, vars []map[string]*querypb.BindVariable, session *SafeSession, autocommit bool, callback func(reply *sqltypes.Result) error) []error
StreamExecuteMulti implements the IExecutor interface
func (*Executor) VSchemaStats ¶
func (e *Executor) VSchemaStats() *VSchemaStats
VSchemaStats returns the loaded vschema stats.
func (*Executor) WriteScatterStats ¶
func (e *Executor) WriteScatterStats(w http.ResponseWriter)
WriteScatterStats will write an html report to the provided response writer
type RegisterVTGate ¶
type RegisterVTGate func(vtgateservice.VTGateService)
RegisterVTGate defines the type of registration mechanism.
type Resolver ¶
type Resolver struct {
// contains filtered or unexported fields
}
Resolver is the layer to resolve KeyspaceIds and KeyRanges to shards. It will try to re-resolve shards if ScatterConn returns retryable error, which may imply horizontal or vertical resharding happened. It is implemented using a srvtopo.Resolver.
func NewResolver ¶
func NewResolver(resolver *srvtopo.Resolver, serv srvtopo.Server, cell string, sc *ScatterConn) *Resolver
NewResolver creates a new Resolver.
func (*Resolver) GetGatewayCacheStatus ¶
func (res *Resolver) GetGatewayCacheStatus() TabletCacheStatusList
GetGatewayCacheStatus returns a displayable version of the Gateway cache.
type SafeSession ¶
SafeSession is a mutex-protected version of the Session. It is thread-safe if each thread only accesses one shard. (the use pattern is 'Find', if not found, then 'AppendOrUpdate', for a single shard)
func NewAutocommitSession ¶
func NewAutocommitSession(sessn *vtgatepb.Session) *SafeSession
NewAutocommitSession returns a SafeSession based on the original session, but with autocommit enabled.
func NewSafeSession ¶
func NewSafeSession(sessn *vtgatepb.Session) *SafeSession
NewSafeSession returns a new SafeSession based on the Session
func (*SafeSession) AddAdvisoryLock ¶ added in v0.14.0
func (session *SafeSession) AddAdvisoryLock(name string)
AddAdvisoryLock adds the advisory lock to the list.
func (*SafeSession) AppendOrUpdate ¶
func (session *SafeSession) AppendOrUpdate(shardSession *vtgatepb.Session_ShardSession, txMode vtgatepb.TransactionMode) error
AppendOrUpdate adds a new ShardSession, or updates an existing one if one already exists for the given shard session
func (*SafeSession) AutocommitApproval ¶
func (session *SafeSession) AutocommitApproval() bool
AutocommitApproval returns true if we can perform a single round-trip autocommit. If so, the caller is responsible for committing their transaction.
func (*SafeSession) CanAddSavepoint ¶ added in v0.13.2
func (session *SafeSession) CanAddSavepoint() bool
CanAddSavepoint returns true if we should insert savepoint and there is no existing savepoint.
func (*SafeSession) ClearAdvisoryLock ¶ added in v0.14.0
func (session *SafeSession) ClearAdvisoryLock()
ClearAdvisoryLock clears the advisory lock list.
func (*SafeSession) ClearWarnings ¶
func (session *SafeSession) ClearWarnings()
ClearWarnings removes all the warnings from the session
func (*SafeSession) EnableLogging ¶ added in v0.15.0
func (session *SafeSession) EnableLogging()
func (*SafeSession) FindAndChangeSessionIfInSingleTxMode ¶ added in v0.15.0
func (session *SafeSession) FindAndChangeSessionIfInSingleTxMode(keyspace, shard string, tabletType topodatapb.TabletType, txMode vtgatepb.TransactionMode) (int64, int64, *topodatapb.TabletAlias, error)
FindAndChangeSessionIfInSingleTxMode returns the transactionId and tabletAlias, if any, for a session modifies the shard session in a specific case for single mode transaction.
func (*SafeSession) GetDDLStrategy ¶ added in v0.9.0
func (session *SafeSession) GetDDLStrategy() string
GetDDLStrategy returns the DDLStrategy value.
func (*SafeSession) GetOrCreateOptions ¶ added in v0.8.0
func (session *SafeSession) GetOrCreateOptions() *querypb.ExecuteOptions
GetOrCreateOptions will return the current options struct, or create one and return it if no-one exists
func (*SafeSession) GetPrepareData ¶ added in v0.17.0
func (session *SafeSession) GetPrepareData(name string) *vtgatepb.PrepareData
GetPrepareData returns the prepared data information for the given key.
func (*SafeSession) GetQueryTimeout ¶ added in v0.16.0
func (session *SafeSession) GetQueryTimeout() int64
GetQueryTimeout gets the query timeout
func (*SafeSession) GetSessionEnableSystemSettings ¶ added in v0.10.0
func (session *SafeSession) GetSessionEnableSystemSettings() bool
GetSessionEnableSystemSettings returns the SessionEnableSystemSettings value.
func (*SafeSession) GetSessionUUID ¶ added in v0.9.0
func (session *SafeSession) GetSessionUUID() string
GetSessionUUID returns the SessionUUID value.
func (*SafeSession) GetSystemVariables ¶ added in v0.14.0
func (session *SafeSession) GetSystemVariables(f func(k string, v string))
GetSystemVariables takes a visitor function that will receive each MySQL system variable in the session. This function will only yield system variables which apply to MySQL itself; Vitess-aware system variables will be skipped.
func (*SafeSession) GetUDV ¶ added in v0.17.0
func (session *SafeSession) GetUDV(name string) *querypb.BindVariable
GetUDV returns the bind variable value for the user defined variable.
func (*SafeSession) HasAdvisoryLock ¶ added in v0.14.0
func (session *SafeSession) HasAdvisoryLock() bool
HasAdvisoryLock returns if any advisory lock is taken
func (*SafeSession) HasSystemVariables ¶ added in v0.14.0
func (session *SafeSession) HasSystemVariables() (found bool)
HasSystemVariables returns whether the session has system variables that would apply to MySQL
func (*SafeSession) InLockSession ¶ added in v0.8.0
func (session *SafeSession) InLockSession() bool
InLockSession returns whether locking is used on this session.
func (*SafeSession) InReservedConn ¶
func (session *SafeSession) InReservedConn() bool
InReservedConn returns true if the session needs to execute on a dedicated connection
func (*SafeSession) InTransaction ¶
func (session *SafeSession) InTransaction() bool
InTransaction returns true if we are in a transaction
func (*SafeSession) IsRollbackSet ¶ added in v0.13.2
func (session *SafeSession) IsRollbackSet() bool
IsRollbackSet returns true if rollback to savepoint can be done.
func (*SafeSession) MustRollback ¶
func (session *SafeSession) MustRollback() bool
MustRollback returns true if the transaction must be rolled back.
func (*SafeSession) RecordWarning ¶
func (session *SafeSession) RecordWarning(warning *querypb.QueryWarning)
RecordWarning stores the given warning in the session
func (*SafeSession) RemoveAdvisoryLock ¶ added in v0.14.0
func (session *SafeSession) RemoveAdvisoryLock(name string)
RemoveAdvisoryLock removes the advisory lock from the list.
func (*SafeSession) RemoveInternalSavepoint ¶ added in v0.14.0
func (session *SafeSession) RemoveInternalSavepoint()
func (*SafeSession) ResetAll ¶ added in v0.8.0
func (session *SafeSession) ResetAll()
ResetAll resets the shard sessions and lock session.
func (*SafeSession) ResetLock ¶ added in v0.8.0
func (session *SafeSession) ResetLock()
ResetLock resets the lock session
func (*SafeSession) ResetShard ¶ added in v0.8.0
func (session *SafeSession) ResetShard(tabletAlias *topodatapb.TabletAlias) error
ResetShard reset the shard session for the provided tablet alias.
func (*SafeSession) SavePoints ¶ added in v0.13.0
func (session *SafeSession) SavePoints() []string
SavePoints returns the save points of the session. It's safe to use concurrently
func (*SafeSession) SavepointRollback ¶ added in v0.13.2
func (session *SafeSession) SavepointRollback()
SavepointRollback updates the state that transaction was rolledback to the savepoint stored in the session.
func (*SafeSession) SetAutocommittable ¶
func (session *SafeSession) SetAutocommittable(flag bool)
SetAutocommittable sets the state to autocommitable if true. Otherwise, it's notAutocommitable.
func (*SafeSession) SetCommitOrder ¶
func (session *SafeSession) SetCommitOrder(co vtgatepb.CommitOrder)
SetCommitOrder sets the commit order.
func (*SafeSession) SetDDLStrategy ¶ added in v0.9.0
func (session *SafeSession) SetDDLStrategy(strategy string)
SetDDLStrategy set the DDLStrategy setting.
func (*SafeSession) SetLockSession ¶ added in v0.8.0
func (session *SafeSession) SetLockSession(lockSession *vtgatepb.Session_ShardSession)
SetLockSession sets the lock session.
func (*SafeSession) SetOptions ¶
func (session *SafeSession) SetOptions(options *querypb.ExecuteOptions)
SetOptions sets the options
func (*SafeSession) SetPreQueries ¶
func (session *SafeSession) SetPreQueries() []string
SetPreQueries returns the prequeries that need to be run when reserving a connection
func (*SafeSession) SetQueryTimeout ¶ added in v0.16.0
func (session *SafeSession) SetQueryTimeout(queryTimeout int64)
SetQueryTimeout sets the query timeout
func (*SafeSession) SetReadAfterWriteGTID ¶ added in v0.9.0
func (session *SafeSession) SetReadAfterWriteGTID(vtgtid string)
SetReadAfterWriteGTID set the ReadAfterWriteGtid setting.
func (*SafeSession) SetReadAfterWriteTimeout ¶ added in v0.9.0
func (session *SafeSession) SetReadAfterWriteTimeout(timeout float64)
SetReadAfterWriteTimeout set the ReadAfterWriteTimeout setting.
func (*SafeSession) SetReservedConn ¶
func (session *SafeSession) SetReservedConn(reservedConn bool)
SetReservedConn set the InReservedConn setting.
func (*SafeSession) SetRollback ¶
func (session *SafeSession) SetRollback()
SetRollback sets the flag indicating that the transaction must be rolled back. The call is a no-op if the session is not in a transaction.
func (*SafeSession) SetRollbackCommand ¶ added in v0.13.2
func (session *SafeSession) SetRollbackCommand()
SetRollbackCommand stores the rollback command to session and executed if required.
func (*SafeSession) SetSavepoint ¶ added in v0.13.2
func (session *SafeSession) SetSavepoint(name string)
SetSavepoint stores the savepoint name to session.
func (*SafeSession) SetSavepointState ¶ added in v0.13.0
func (session *SafeSession) SetSavepointState(spNeed bool)
SetSavepointState sets the state only once for the complete query execution life. Calling the function multiple times will have no effect, only the first call would be used. Default state is savepointStateNotSet, if savepoint needed (spNeed true) then it will be set to savepointNeeded otherwise savepointNotNeeded.
func (*SafeSession) SetSessionEnableSystemSettings ¶ added in v0.10.0
func (session *SafeSession) SetSessionEnableSystemSettings(allow bool)
SetSessionEnableSystemSettings set the SessionEnableSystemSettings setting.
func (*SafeSession) SetSessionTrackGtids ¶ added in v0.9.0
func (session *SafeSession) SetSessionTrackGtids(enable bool)
SetSessionTrackGtids set the SessionTrackGtids setting.
func (*SafeSession) SetSystemVariable ¶
func (session *SafeSession) SetSystemVariable(name string, expr string)
SetSystemVariable sets the system variable in the session.
func (*SafeSession) SetTargetString ¶
func (session *SafeSession) SetTargetString(target string)
SetTargetString sets the target string in the session.
func (*SafeSession) SetUserDefinedVariable ¶
func (session *SafeSession) SetUserDefinedVariable(key string, value *querypb.BindVariable)
SetUserDefinedVariable sets the user defined variable in the session.
func (*SafeSession) StorePrepareData ¶ added in v0.17.0
func (session *SafeSession) StorePrepareData(key string, value *vtgatepb.PrepareData)
StorePrepareData stores the prepared data information for the given key.
func (*SafeSession) StoreSavepoint ¶
func (session *SafeSession) StoreSavepoint(sql string)
StoreSavepoint stores the savepoint and release savepoint queries in the session
func (*SafeSession) TimeZone ¶ added in v0.17.0
func (session *SafeSession) TimeZone() *time.Location
func (*SafeSession) TriggerLockHeartBeat ¶ added in v0.8.0
func (session *SafeSession) TriggerLockHeartBeat() bool
TriggerLockHeartBeat returns if it time to trigger next lock heartbeat
func (*SafeSession) UpdateLockHeartbeat ¶ added in v0.8.0
func (session *SafeSession) UpdateLockHeartbeat()
UpdateLockHeartbeat updates the LastLockHeartbeat time
type ScatterConn ¶
type ScatterConn struct {
// contains filtered or unexported fields
}
ScatterConn is used for executing queries across multiple shard level connections.
func NewScatterConn ¶
func NewScatterConn(statsName string, txConn *TxConn, gw *TabletGateway) *ScatterConn
NewScatterConn creates a new ScatterConn.
func (*ScatterConn) Close ¶
func (stc *ScatterConn) Close() error
Close closes the underlying Gateway.
func (*ScatterConn) ExecuteLock ¶ added in v0.8.0
func (stc *ScatterConn) ExecuteLock(ctx context.Context, rs *srvtopo.ResolvedShard, query *querypb.BoundQuery, session *SafeSession, lockFuncType sqlparser.LockingFuncType) (*sqltypes.Result, error)
ExecuteLock performs the requested 'action' on the specified ResolvedShard. If the lock session already has a reserved connection, it reuses it. Otherwise open a new reserved connection. The action function must match the shardActionTransactionFunc signature.
It returns an error recorder in which each shard error is recorded positionally, i.e. if rss[2] had an error, then the error recorder will store that error in the second position.
func (*ScatterConn) ExecuteMultiShard ¶
func (stc *ScatterConn) ExecuteMultiShard( ctx context.Context, primitive engine.Primitive, rss []*srvtopo.ResolvedShard, queries []*querypb.BoundQuery, session *SafeSession, autocommit bool, ignoreMaxMemoryRows bool, ) (qr *sqltypes.Result, errs []error)
ExecuteMultiShard is like Execute, but each shard gets its own Sql Queries and BindVariables.
It always returns a non-nil query result and an array of shard errors which may be nil so that callers can optionally process a partially-successful operation.
func (*ScatterConn) GetGatewayCacheStatus ¶
func (stc *ScatterConn) GetGatewayCacheStatus() TabletCacheStatusList
GetGatewayCacheStatus returns a displayable version of the Gateway cache.
func (*ScatterConn) GetHealthCheckCacheStatus ¶
func (stc *ScatterConn) GetHealthCheckCacheStatus() discovery.TabletsCacheStatusList
GetHealthCheckCacheStatus returns a displayable version of the HealthCheck cache.
func (*ScatterConn) MessageStream ¶
func (stc *ScatterConn) MessageStream(ctx context.Context, rss []*srvtopo.ResolvedShard, name string, callback func(*sqltypes.Result) error) error
MessageStream streams messages from the specified shards. Note we guarantee the callback will not be called concurrently by multiple go routines, through processOneStreamingResult.
func (*ScatterConn) StreamExecuteMulti ¶
func (stc *ScatterConn) StreamExecuteMulti( ctx context.Context, primitive engine.Primitive, query string, rss []*srvtopo.ResolvedShard, bindVars []map[string]*querypb.BindVariable, session *SafeSession, autocommit bool, callback func(reply *sqltypes.Result) error, ) []error
StreamExecuteMulti is like StreamExecute, but each shard gets its own bindVars. If len(shards) is not equal to len(bindVars), the function panics. Note we guarantee the callback will not be called concurrently by multiple go routines, through processOneStreamingResult.
type SchemaInfo ¶ added in v0.11.0
type SchemaInfo interface { Tables(ks string) map[string][]vindexes.Column Views(ks string) map[string]sqlparser.SelectStatement }
SchemaInfo is an interface to schema tracker.
type TabletCacheStatus ¶
type TabletCacheStatus struct { Keyspace string Shard string TabletType topodatapb.TabletType Name string Addr string QueryCount uint64 QueryError uint64 QPS float64 AvgLatency float64 // in milliseconds }
TabletCacheStatus contains the status per destination for a gateway.
func (*TabletCacheStatus) FormattedQPS ¶
func (tcs *TabletCacheStatus) FormattedQPS() string
FormattedQPS shows a 2 digit rounded value of QPS. Used in the HTML template above.
type TabletCacheStatusList ¶
type TabletCacheStatusList []*TabletCacheStatus
TabletCacheStatusList is a slice of TabletCacheStatus.
func (TabletCacheStatusList) Len ¶
func (gtcsl TabletCacheStatusList) Len() int
Len is part of sort.Interface.
func (TabletCacheStatusList) Less ¶
func (gtcsl TabletCacheStatusList) Less(i, j int) bool
Less is part of sort.Interface.
func (TabletCacheStatusList) Swap ¶
func (gtcsl TabletCacheStatusList) Swap(i, j int)
Swap is part of sort.Interface.
type TabletGateway ¶
type TabletGateway struct { queryservice.QueryService // contains filtered or unexported fields }
TabletGateway implements the Gateway interface. This implementation uses the new healthcheck module.
func NewTabletGateway ¶
func NewTabletGateway(ctx context.Context, hc discovery.HealthCheck, serv srvtopo.Server, localCell string) *TabletGateway
NewTabletGateway creates and returns a new TabletGateway
func (*TabletGateway) CacheStatus ¶
func (gw *TabletGateway) CacheStatus() TabletCacheStatusList
CacheStatus returns a list of TabletCacheStatus per keyspace/shard/tablet_type.
func (*TabletGateway) Close ¶
func (gw *TabletGateway) Close(_ context.Context) error
Close shuts down underlying connections. This function hides the inner implementation.
func (*TabletGateway) DefaultConnCollation ¶ added in v0.13.0
func (gw *TabletGateway) DefaultConnCollation() collations.ID
DefaultConnCollation returns the default connection collation of this TabletGateway
func (*TabletGateway) QueryServiceByAlias ¶
func (gw *TabletGateway) QueryServiceByAlias(alias *topodatapb.TabletAlias, target *querypb.Target) (queryservice.QueryService, error)
QueryServiceByAlias satisfies the Gateway interface
func (*TabletGateway) RegisterStats ¶
func (gw *TabletGateway) RegisterStats()
RegisterStats registers the stats to export the lag since the last refresh and the checksum of the topology
func (*TabletGateway) TabletsCacheStatus ¶
func (gw *TabletGateway) TabletsCacheStatus() discovery.TabletsCacheStatusList
TabletsCacheStatus returns a displayable version of the health check cache.
func (*TabletGateway) WaitForTablets ¶
func (gw *TabletGateway) WaitForTablets(tabletTypesToWait []topodatapb.TabletType) (err error)
WaitForTablets is part of the Gateway interface.
type TabletStatusAggregator ¶
type TabletStatusAggregator struct { Keyspace string Shard string TabletType topodatapb.TabletType Name string // the alternative name of a tablet Addr string // the host:port of a tablet QueryCount uint64 QueryError uint64 // contains filtered or unexported fields }
TabletStatusAggregator tracks tablet status for a gateway.
func NewTabletStatusAggregator ¶
func NewTabletStatusAggregator(keyspace, shard string, tabletType topodatapb.TabletType, name string) *TabletStatusAggregator
NewTabletStatusAggregator creates a TabletStatusAggregator.
func (*TabletStatusAggregator) GetCacheStatus ¶
func (tsa *TabletStatusAggregator) GetCacheStatus() *TabletCacheStatus
GetCacheStatus returns a TabletCacheStatus representing the current gateway status.
func (*TabletStatusAggregator) UpdateQueryInfo ¶
func (tsa *TabletStatusAggregator) UpdateQueryInfo(addr string, tabletType topodatapb.TabletType, elapsed time.Duration, hasError bool)
UpdateQueryInfo updates the aggregator with the given information about a query.
type TxConn ¶
type TxConn struct {
// contains filtered or unexported fields
}
TxConn is used for executing transactional requests.
func NewTxConn ¶
func NewTxConn(gw *TabletGateway, txMode vtgatepb.TransactionMode) *TxConn
NewTxConn builds a new TxConn.
func (*TxConn) Begin ¶
func (txc *TxConn) Begin(ctx context.Context, session *SafeSession, txAccessModes []sqlparser.TxAccessMode) error
Begin begins a new transaction. If one is already in progress, it commits it and starts a new one.
func (*TxConn) Commit ¶
func (txc *TxConn) Commit(ctx context.Context, session *SafeSession) error
Commit commits the current transaction. The type of commit can be best effort or 2pc depending on the session setting.
func (*TxConn) Release ¶
func (txc *TxConn) Release(ctx context.Context, session *SafeSession) error
Release releases the reserved connection and/or rollbacks the transaction
func (*TxConn) ReleaseAll ¶ added in v0.8.0
func (txc *TxConn) ReleaseAll(ctx context.Context, session *SafeSession) error
ReleaseAll releases all the shard sessions and lock session.
func (*TxConn) ReleaseLock ¶ added in v0.8.0
func (txc *TxConn) ReleaseLock(ctx context.Context, session *SafeSession) error
ReleaseLock releases the reserved connection used for locking.
type VSchemaKeyspaceStats ¶
type VSchemaKeyspaceStats struct { Keyspace string Sharded bool TableCount int VindexCount int Error string }
VSchemaKeyspaceStats contains a rollup of the VSchema stats for a keyspace. It is used to display a table with the information in the status page.
type VSchemaManager ¶
type VSchemaManager struct {
// contains filtered or unexported fields
}
VSchemaManager is used to watch for updates to the vschema and to implement the DDL commands to add / remove vindexes
func (*VSchemaManager) GetCurrentSrvVschema ¶
func (vm *VSchemaManager) GetCurrentSrvVschema() *vschemapb.SrvVSchema
GetCurrentSrvVschema returns a copy of the latest SrvVschema from the topo watch
func (*VSchemaManager) Rebuild ¶ added in v0.11.0
func (vm *VSchemaManager) Rebuild()
Rebuild will rebuild and publish the new vschema. This method should be called when the underlying schema has changed.
func (*VSchemaManager) UpdateVSchema ¶
func (vm *VSchemaManager) UpdateVSchema(ctx context.Context, ksName string, vschema *vschemapb.SrvVSchema) error
UpdateVSchema propagates the updated vschema to the topo. The entry for the given keyspace is updated in the global topo, and the full SrvVSchema is updated in all known cells.
func (*VSchemaManager) VSchemaUpdate ¶ added in v0.11.0
func (vm *VSchemaManager) VSchemaUpdate(v *vschemapb.SrvVSchema, err error) bool
VSchemaUpdate builds the VSchema from SrvVschema and call subscribers.
type VSchemaOperator ¶
type VSchemaOperator interface { GetCurrentSrvVschema() *vschemapb.SrvVSchema UpdateVSchema(ctx context.Context, ksName string, vschema *vschemapb.SrvVSchema) error }
VSchemaOperator is an interface to Vschema Operations
type VSchemaStats ¶
type VSchemaStats struct { Error string Keyspaces []*VSchemaKeyspaceStats }
VSchemaStats contains a rollup of the VSchema stats.
func NewVSchemaStats ¶
func NewVSchemaStats(vschema *vindexes.VSchema, errorMessage string) *VSchemaStats
NewVSchemaStats returns a new VSchemaStats from a VSchema.
type VTGate ¶
type VTGate struct {
// contains filtered or unexported fields
}
VTGate is the rpc interface to vtgate. Only one instance can be created. It implements vtgateservice.VTGateService VTGate exposes multiple generations of interfaces.
func Init ¶
func Init( ctx context.Context, hc discovery.HealthCheck, serv srvtopo.Server, cell string, tabletTypesToWait []topodatapb.TabletType, pv plancontext.PlannerVersion, ) *VTGate
Init initializes VTGate server.
func (*VTGate) CloseSession ¶
CloseSession closes the session, rolling back any implicit transactions. This has the same effect as if a "rollback" statement was executed, but does not affect the query statistics.
func (*VTGate) Execute ¶
func (vtg *VTGate) Execute(ctx context.Context, session *vtgatepb.Session, sql string, bindVariables map[string]*querypb.BindVariable) (newSession *vtgatepb.Session, qr *sqltypes.Result, err error)
Execute executes a non-streaming query. This is a V3 function.
func (*VTGate) ExecuteBatch ¶
func (vtg *VTGate) ExecuteBatch(ctx context.Context, session *vtgatepb.Session, sqlList []string, bindVariablesList []map[string]*querypb.BindVariable) (*vtgatepb.Session, []sqltypes.QueryResponse, error)
ExecuteBatch executes a batch of queries. This is a V3 function.
func (*VTGate) Gateway ¶
func (vtg *VTGate) Gateway() *TabletGateway
Gateway returns the current gateway implementation. Mostly used for tests.
func (*VTGate) GetGatewayCacheStatus ¶
func (vtg *VTGate) GetGatewayCacheStatus() TabletCacheStatusList
GetGatewayCacheStatus returns a displayable version of the Gateway cache.
func (*VTGate) HandlePanic ¶
HandlePanic recovers from panics, and logs / increment counters
func (*VTGate) IsHealthy ¶
IsHealthy returns nil if server is healthy. Otherwise, it returns an error indicating the reason.
func (*VTGate) Prepare ¶
func (vtg *VTGate) Prepare(ctx context.Context, session *vtgatepb.Session, sql string, bindVariables map[string]*querypb.BindVariable) (newSession *vtgatepb.Session, fld []*querypb.Field, err error)
Prepare supports non-streaming prepare statement query with multi shards
func (*VTGate) ResolveTransaction ¶
ResolveTransaction resolves the specified 2PC transaction.
func (*VTGate) StreamExecute ¶
func (vtg *VTGate) StreamExecute(ctx context.Context, session *vtgatepb.Session, sql string, bindVariables map[string]*querypb.BindVariable, callback func(*sqltypes.Result) error) (*vtgatepb.Session, error)
StreamExecute executes a streaming query. This is a V3 function. Note we guarantee the callback will not be called concurrently by multiple go routines.
func (*VTGate) VSchemaStats ¶
func (vtg *VTGate) VSchemaStats() *VSchemaStats
VSchemaStats returns the loaded vschema stats.
func (*VTGate) VStream ¶
func (vtg *VTGate) VStream(ctx context.Context, tabletType topodatapb.TabletType, vgtid *binlogdatapb.VGtid, filter *binlogdatapb.Filter, flags *vtgatepb.VStreamFlags, send func([]*binlogdatapb.VEvent) error) error
VStream streams binlog events.
Source Files ¶
- api.go
- debugenv.go
- executor.go
- executor_scatter_stats.go
- executor_stats.go
- plan_execute.go
- plugin_mysql_server.go
- querylog.go
- querylogz.go
- queryz.go
- resolver.go
- safe_session.go
- scatter_conn.go
- status.go
- tabletgateway.go
- tx_conn.go
- vcursor_impl.go
- vschema_manager.go
- vschema_stats.go
- vstream_manager.go
- vtgate.go
Directories ¶
Path | Synopsis |
---|---|
Package buffer provides a buffer for PRIMARY traffic during failovers.
|
Package buffer provides a buffer for PRIMARY traffic during failovers. |
Package fakerpcvtgateconn provides a fake implementation of vtgateconn.Impl that doesn't do any RPC, but uses a local map to return results.
|
Package fakerpcvtgateconn provides a fake implementation of vtgateconn.Impl that doesn't do any RPC, but uses a local map to return results. |
Package grpcvtgateconn provides gRPC connectivity for VTGate.
|
Package grpcvtgateconn provides gRPC connectivity for VTGate. |
Package grpcvtgateservice provides the gRPC glue for vtgate
|
Package grpcvtgateservice provides the gRPC glue for vtgate |
Package planbuilder allows you to build execution plans that describe how to fulfill a query that may span multiple keyspaces or shards.
|
Package planbuilder allows you to build execution plans that describe how to fulfill a query that may span multiple keyspaces or shards. |
operators
Package operators contains the operators used to plan queries.
|
Package operators contains the operators used to plan queries. |
Package vtgateservice provides to interface definition for the vtgate service
|
Package vtgateservice provides to interface definition for the vtgate service |