Commit 98681675 authored by Amit Kapila's avatar Amit Kapila

Track statistics for spilling of changes from ReorderBuffer.

This adds the statistics about transactions spilled to disk from
ReorderBuffer. Users can query the pg_stat_replication_slots view to check
these stats and call pg_stat_reset_replication_slot to reset the stats of
a particular slot. Users can pass NULL in pg_stat_reset_replication_slot
to reset stats of all the slots.

This commit extends the statistics collector to track this information
about slots.

Author: Sawada Masahiko and Amit Kapila
Reviewed-by: Amit Kapila and Dilip Kumar
Discussion: https://postgr.es/m/CA+fd4k5_pPAYRTDrO2PbtTOe0eHQpBvuqmCr8ic39uTNmR49Eg@mail.gmail.com
parent 8d2a01ae
...@@ -314,6 +314,15 @@ postgres 27093 0.0 0.0 30096 2752 ? Ss 11:34 0:00 postgres: ser ...@@ -314,6 +314,15 @@ postgres 27093 0.0 0.0 30096 2752 ? Ss 11:34 0:00 postgres: ser
</entry> </entry>
</row> </row>
<row>
<entry><structname>pg_stat_replication_slots</structname><indexterm><primary>pg_stat_replication_slots</primary></indexterm></entry>
<entry>One row per replication slot, showing statistics about
replication slot usage.
See <link linkend="monitoring-pg-stat-replication-slots-view">
<structname>pg_stat_replication_slots</structname></link> for details.
</entry>
</row>
<row> <row>
<entry><structname>pg_stat_wal_receiver</structname><indexterm><primary>pg_stat_wal_receiver</primary></indexterm></entry> <entry><structname>pg_stat_wal_receiver</structname><indexterm><primary>pg_stat_wal_receiver</primary></indexterm></entry>
<entry>Only one row, showing statistics about the WAL receiver from <entry>Only one row, showing statistics about the WAL receiver from
...@@ -2552,6 +2561,88 @@ SELECT pid, wait_event_type, wait_event FROM pg_stat_activity WHERE wait_event i ...@@ -2552,6 +2561,88 @@ SELECT pid, wait_event_type, wait_event FROM pg_stat_activity WHERE wait_event i
</sect2> </sect2>
<sect2 id="monitoring-pg-stat-replication-slots-view">
<title><structname>pg_stat_replication_slots</structname></title>
<indexterm>
<primary>pg_stat_replication_slots</primary>
</indexterm>
<para>
The <structname>pg_stat_replication_slots</structname> view will contain
one row per logical replication slot, showing statistics about its usage.
</para>
<table id="pg-stat-replication-slots-view" xreflabel="pg_stat_replication_slots">
<title><structname>pg_stat_replication_slots</structname> View</title>
<tgroup cols="1">
<thead>
<row>
<entry role="catalog_table_entry"><para role="column_definition">
Column Type
</para>
<para>
Description
</para></entry>
</row>
</thead>
<tbody>
<row>
<entry role="catalog_table_entry"><para role="column_definition">
<structfield>name</structfield> <type>text</type>
</para>
<para>
A unique, cluster-wide identifier for the replication slot
</para></entry>
</row>
<row>
<entry role="catalog_table_entry"><para role="column_definition">
<structfield>spill_txns</structfield> <type>bigint</type>
</para>
<para>
Number of transactions spilled to disk after the memory used by
logical decoding exceeds <literal>logical_decoding_work_mem</literal>. The
counter gets incremented both for toplevel transactions and
subtransactions.
</para></entry>
</row>
<row>
<entry role="catalog_table_entry"><para role="column_definition">
<structfield>spill_count</structfield> <type>bigint</type>
</para>
<para>
Number of times transactions were spilled to disk. Transactions
may get spilled repeatedly, and this counter gets incremented on every
such invocation.
</para></entry>
</row>
<row>
<entry role="catalog_table_entry"><para role="column_definition">
<structfield>spill_bytes</structfield> <type>bigint</type>
</para>
<para>
Amount of decoded transaction data spilled to disk.
</para></entry>
</row>
<row>
<entry role="catalog_table_entry"><para role="column_definition">
<structfield>stats_reset</structfield> <type>timestamp with time zone</type>
</para>
<para>
Time at which these statistics were last reset
</para></entry>
</row>
</tbody>
</tgroup>
</table>
</sect2>
<sect2 id="monitoring-pg-stat-wal-receiver-view"> <sect2 id="monitoring-pg-stat-wal-receiver-view">
<title><structname>pg_stat_wal_receiver</structname></title> <title><structname>pg_stat_wal_receiver</structname></title>
...@@ -4802,6 +4893,27 @@ SELECT pid, wait_event_type, wait_event FROM pg_stat_activity WHERE wait_event i ...@@ -4802,6 +4893,27 @@ SELECT pid, wait_event_type, wait_event FROM pg_stat_activity WHERE wait_event i
can be granted EXECUTE to run the function. can be granted EXECUTE to run the function.
</para></entry> </para></entry>
</row> </row>
<row>
<entry role="func_table_entry"><para role="func_signature">
<indexterm>
<primary>pg_stat_reset_replication_slot</primary>
</indexterm>
<function>pg_stat_reset_replication_slot</function> ( <type>text</type> )
<returnvalue>void</returnvalue>
</para>
<para>
Resets statistics to zero for a single replication slot, or for all
replication slots in the cluster. The argument can be either the name
of the slot to reset the stats or NULL. If the argument is NULL, all
counters shown in the <structname>pg_stat_replication_slots</structname>
view for all replication slots are reset.
</para>
<para>
This function is restricted to superusers by default, but other users
can be granted EXECUTE to run the function.
</para></entry>
</row>
</tbody> </tbody>
</tgroup> </tgroup>
</table> </table>
......
...@@ -796,6 +796,15 @@ CREATE VIEW pg_stat_replication AS ...@@ -796,6 +796,15 @@ CREATE VIEW pg_stat_replication AS
JOIN pg_stat_get_wal_senders() AS W ON (S.pid = W.pid) JOIN pg_stat_get_wal_senders() AS W ON (S.pid = W.pid)
LEFT JOIN pg_authid AS U ON (S.usesysid = U.oid); LEFT JOIN pg_authid AS U ON (S.usesysid = U.oid);
CREATE VIEW pg_stat_replication_slots AS
SELECT
s.name,
s.spill_txns,
s.spill_count,
s.spill_bytes,
s.stats_reset
FROM pg_stat_get_replication_slots() AS s;
CREATE VIEW pg_stat_slru AS CREATE VIEW pg_stat_slru AS
SELECT SELECT
s.name, s.name,
...@@ -1453,6 +1462,7 @@ REVOKE EXECUTE ON FUNCTION pg_stat_reset_shared(text) FROM public; ...@@ -1453,6 +1462,7 @@ REVOKE EXECUTE ON FUNCTION pg_stat_reset_shared(text) FROM public;
REVOKE EXECUTE ON FUNCTION pg_stat_reset_slru(text) FROM public; REVOKE EXECUTE ON FUNCTION pg_stat_reset_slru(text) FROM public;
REVOKE EXECUTE ON FUNCTION pg_stat_reset_single_table_counters(oid) FROM public; REVOKE EXECUTE ON FUNCTION pg_stat_reset_single_table_counters(oid) FROM public;
REVOKE EXECUTE ON FUNCTION pg_stat_reset_single_function_counters(oid) FROM public; REVOKE EXECUTE ON FUNCTION pg_stat_reset_single_function_counters(oid) FROM public;
REVOKE EXECUTE ON FUNCTION pg_stat_reset_replication_slot(text) FROM public;
REVOKE EXECUTE ON FUNCTION lo_import(text) FROM public; REVOKE EXECUTE ON FUNCTION lo_import(text) FROM public;
REVOKE EXECUTE ON FUNCTION lo_import(text, oid) FROM public; REVOKE EXECUTE ON FUNCTION lo_import(text, oid) FROM public;
......
This diff is collapsed.
...@@ -650,6 +650,12 @@ DecodeCommit(LogicalDecodingContext *ctx, XLogRecordBuffer *buf, ...@@ -650,6 +650,12 @@ DecodeCommit(LogicalDecodingContext *ctx, XLogRecordBuffer *buf,
/* replay actions of all transaction + subtransactions in order */ /* replay actions of all transaction + subtransactions in order */
ReorderBufferCommit(ctx->reorder, xid, buf->origptr, buf->endptr, ReorderBufferCommit(ctx->reorder, xid, buf->origptr, buf->endptr,
commit_time, origin_id, origin_lsn); commit_time, origin_id, origin_lsn);
/*
* Update the decoding stats at transaction commit/abort. It is not clear
* that sending more or less frequently than this would be better.
*/
UpdateDecodingStats(ctx);
} }
/* /*
...@@ -669,6 +675,9 @@ DecodeAbort(LogicalDecodingContext *ctx, XLogRecordBuffer *buf, ...@@ -669,6 +675,9 @@ DecodeAbort(LogicalDecodingContext *ctx, XLogRecordBuffer *buf,
} }
ReorderBufferAbort(ctx->reorder, xid, buf->record->EndRecPtr); ReorderBufferAbort(ctx->reorder, xid, buf->record->EndRecPtr);
/* update the decoding stats */
UpdateDecodingStats(ctx);
} }
/* /*
......
...@@ -32,6 +32,7 @@ ...@@ -32,6 +32,7 @@
#include "access/xlog_internal.h" #include "access/xlog_internal.h"
#include "fmgr.h" #include "fmgr.h"
#include "miscadmin.h" #include "miscadmin.h"
#include "pgstat.h"
#include "replication/decode.h" #include "replication/decode.h"
#include "replication/logical.h" #include "replication/logical.h"
#include "replication/origin.h" #include "replication/origin.h"
...@@ -1460,3 +1461,31 @@ ResetLogicalStreamingState(void) ...@@ -1460,3 +1461,31 @@ ResetLogicalStreamingState(void)
CheckXidAlive = InvalidTransactionId; CheckXidAlive = InvalidTransactionId;
bsysscan = false; bsysscan = false;
} }
/*
* Report stats for a slot.
*/
void
UpdateDecodingStats(LogicalDecodingContext *ctx)
{
ReorderBuffer *rb = ctx->reorder;
/*
* Nothing to do if we haven't spilled anything since the last time the
* stats has been sent.
*/
if (rb->spillBytes <= 0)
return;
elog(DEBUG2, "UpdateSpillStats: updating stats %p %lld %lld %lld",
rb,
(long long) rb->spillTxns,
(long long) rb->spillCount,
(long long) rb->spillBytes);
pgstat_report_replslot(NameStr(ctx->slot->data.name),
rb->spillTxns, rb->spillCount, rb->spillBytes);
rb->spillTxns = 0;
rb->spillCount = 0;
rb->spillBytes = 0;
}
...@@ -343,6 +343,10 @@ ReorderBufferAllocate(void) ...@@ -343,6 +343,10 @@ ReorderBufferAllocate(void)
buffer->outbufsize = 0; buffer->outbufsize = 0;
buffer->size = 0; buffer->size = 0;
buffer->spillTxns = 0;
buffer->spillCount = 0;
buffer->spillBytes = 0;
buffer->current_restart_decoding_lsn = InvalidXLogRecPtr; buffer->current_restart_decoding_lsn = InvalidXLogRecPtr;
dlist_init(&buffer->toplevel_by_lsn); dlist_init(&buffer->toplevel_by_lsn);
...@@ -1579,6 +1583,13 @@ ReorderBufferTruncateTXN(ReorderBuffer *rb, ReorderBufferTXN *txn) ...@@ -1579,6 +1583,13 @@ ReorderBufferTruncateTXN(ReorderBuffer *rb, ReorderBufferTXN *txn)
{ {
ReorderBufferRestoreCleanup(rb, txn); ReorderBufferRestoreCleanup(rb, txn);
txn->txn_flags &= ~RBTXN_IS_SERIALIZED; txn->txn_flags &= ~RBTXN_IS_SERIALIZED;
/*
* We set this flag to indicate if the transaction is ever serialized.
* We need this to accurately update the stats as otherwise the same
* transaction can be counted as serialized multiple times.
*/
txn->txn_flags |= RBTXN_IS_SERIALIZED_CLEAR;
} }
/* also reset the number of entries in the transaction */ /* also reset the number of entries in the transaction */
...@@ -3112,6 +3123,7 @@ ReorderBufferSerializeTXN(ReorderBuffer *rb, ReorderBufferTXN *txn) ...@@ -3112,6 +3123,7 @@ ReorderBufferSerializeTXN(ReorderBuffer *rb, ReorderBufferTXN *txn)
int fd = -1; int fd = -1;
XLogSegNo curOpenSegNo = 0; XLogSegNo curOpenSegNo = 0;
Size spilled = 0; Size spilled = 0;
Size size = txn->size;
elog(DEBUG2, "spill %u changes in XID %u to disk", elog(DEBUG2, "spill %u changes in XID %u to disk",
(uint32) txn->nentries_mem, txn->xid); (uint32) txn->nentries_mem, txn->xid);
...@@ -3170,6 +3182,16 @@ ReorderBufferSerializeTXN(ReorderBuffer *rb, ReorderBufferTXN *txn) ...@@ -3170,6 +3182,16 @@ ReorderBufferSerializeTXN(ReorderBuffer *rb, ReorderBufferTXN *txn)
spilled++; spilled++;
} }
/* update the statistics iff we have spilled anything */
if (spilled)
{
rb->spillCount += 1;
rb->spillBytes += size;
/* don't consider already serialized transactions */
rb->spillTxns += (rbtxn_is_serialized(txn) || rbtxn_is_serialized_clear(txn)) ? 0 : 1;
}
Assert(spilled == txn->nentries_mem); Assert(spilled == txn->nentries_mem);
Assert(dlist_is_empty(&txn->changes)); Assert(dlist_is_empty(&txn->changes));
txn->nentries_mem = 0; txn->nentries_mem = 0;
......
...@@ -99,7 +99,6 @@ ReplicationSlot *MyReplicationSlot = NULL; ...@@ -99,7 +99,6 @@ ReplicationSlot *MyReplicationSlot = NULL;
int max_replication_slots = 0; /* the maximum number of replication int max_replication_slots = 0; /* the maximum number of replication
* slots */ * slots */
static ReplicationSlot *SearchNamedReplicationSlot(const char *name);
static int ReplicationSlotAcquireInternal(ReplicationSlot *slot, static int ReplicationSlotAcquireInternal(ReplicationSlot *slot,
const char *name, SlotAcquireBehavior behavior); const char *name, SlotAcquireBehavior behavior);
static void ReplicationSlotDropAcquired(void); static void ReplicationSlotDropAcquired(void);
...@@ -314,6 +313,15 @@ ReplicationSlotCreate(const char *name, bool db_specific, ...@@ -314,6 +313,15 @@ ReplicationSlotCreate(const char *name, bool db_specific,
LWLockRelease(ReplicationSlotControlLock); LWLockRelease(ReplicationSlotControlLock);
/*
* Create statistics entry for the new logical slot. We don't collect any
* stats for physical slots, so no need to create an entry for the same.
* See ReplicationSlotDropPtr for why we need to do this before releasing
* ReplicationSlotAllocationLock.
*/
if (SlotIsLogical(slot))
pgstat_report_replslot(NameStr(slot->data.name), 0, 0, 0);
/* /*
* Now that the slot has been marked as in_use and active, it's safe to * Now that the slot has been marked as in_use and active, it's safe to
* let somebody else try to allocate a slot. * let somebody else try to allocate a slot.
...@@ -331,7 +339,7 @@ ReplicationSlotCreate(const char *name, bool db_specific, ...@@ -331,7 +339,7 @@ ReplicationSlotCreate(const char *name, bool db_specific,
* *
* The caller must hold ReplicationSlotControlLock in shared mode. * The caller must hold ReplicationSlotControlLock in shared mode.
*/ */
static ReplicationSlot * ReplicationSlot *
SearchNamedReplicationSlot(const char *name) SearchNamedReplicationSlot(const char *name)
{ {
int i; int i;
...@@ -683,6 +691,19 @@ ReplicationSlotDropPtr(ReplicationSlot *slot) ...@@ -683,6 +691,19 @@ ReplicationSlotDropPtr(ReplicationSlot *slot)
ereport(WARNING, ereport(WARNING,
(errmsg("could not remove directory \"%s\"", tmppath))); (errmsg("could not remove directory \"%s\"", tmppath)));
/*
* Send a message to drop the replication slot to the stats collector.
* Since there is no guarantee of the order of message transfer on a UDP
* connection, it's possible that a message for creating a new slot
* reaches before a message for removing the old slot. We send the drop
* and create messages while holding ReplicationSlotAllocationLock to
* reduce that possibility. If the messages reached in reverse, we would
* lose one statistics update message. But the next update message will
* create the statistics for the replication slot.
*/
if (SlotIsLogical(slot))
pgstat_report_replslot_drop(NameStr(slot->data.name));
/* /*
* We release this at the very end, so that nobody starts trying to create * We release this at the very end, so that nobody starts trying to create
* a slot while we're still cleaning up the detritus of the old one. * a slot while we're still cleaning up the detritus of the old one.
......
...@@ -2069,6 +2069,20 @@ pg_stat_reset_slru(PG_FUNCTION_ARGS) ...@@ -2069,6 +2069,20 @@ pg_stat_reset_slru(PG_FUNCTION_ARGS)
PG_RETURN_VOID(); PG_RETURN_VOID();
} }
/* Reset replication slots stats (a specific one or all of them). */
Datum
pg_stat_reset_replication_slot(PG_FUNCTION_ARGS)
{
char *target = NULL;
if (!PG_ARGISNULL(0))
target = text_to_cstring(PG_GETARG_TEXT_PP(0));
pgstat_reset_replslot_counter(target);
PG_RETURN_VOID();
}
Datum Datum
pg_stat_get_archiver(PG_FUNCTION_ARGS) pg_stat_get_archiver(PG_FUNCTION_ARGS)
{ {
...@@ -2134,3 +2148,69 @@ pg_stat_get_archiver(PG_FUNCTION_ARGS) ...@@ -2134,3 +2148,69 @@ pg_stat_get_archiver(PG_FUNCTION_ARGS)
/* Returns the record as Datum */ /* Returns the record as Datum */
PG_RETURN_DATUM(HeapTupleGetDatum(heap_form_tuple(tupdesc, values, nulls))); PG_RETURN_DATUM(HeapTupleGetDatum(heap_form_tuple(tupdesc, values, nulls)));
} }
/* Get the statistics for the replication slots */
Datum
pg_stat_get_replication_slots(PG_FUNCTION_ARGS)
{
#define PG_STAT_GET_REPLICATION_SLOT_CLOS 5
ReturnSetInfo *rsinfo = (ReturnSetInfo *) fcinfo->resultinfo;
TupleDesc tupdesc;
Tuplestorestate *tupstore;
MemoryContext per_query_ctx;
MemoryContext oldcontext;
PgStat_ReplSlotStats *slotstats;
int nstats;
int i;
/* check to see if caller supports us returning a tuplestore */
if (rsinfo == NULL || !IsA(rsinfo, ReturnSetInfo))
ereport(ERROR,
(errcode(ERRCODE_FEATURE_NOT_SUPPORTED),
errmsg("set-valued function called in context that cannot accept a set")));
if (!(rsinfo->allowedModes & SFRM_Materialize))
ereport(ERROR,
(errcode(ERRCODE_FEATURE_NOT_SUPPORTED),
errmsg("materialize mode required, but it is not allowed in this context")));
/* Build a tuple descriptor for our result type */
if (get_call_result_type(fcinfo, NULL, &tupdesc) != TYPEFUNC_COMPOSITE)
elog(ERROR, "return type must be a row type");
per_query_ctx = rsinfo->econtext->ecxt_per_query_memory;
oldcontext = MemoryContextSwitchTo(per_query_ctx);
tupstore = tuplestore_begin_heap(true, false, work_mem);
rsinfo->returnMode = SFRM_Materialize;
rsinfo->setResult = tupstore;
rsinfo->setDesc = tupdesc;
MemoryContextSwitchTo(oldcontext);
slotstats = pgstat_fetch_replslot(&nstats);
for (i = 0; i < nstats; i++)
{
Datum values[PG_STAT_GET_REPLICATION_SLOT_CLOS];
bool nulls[PG_STAT_GET_REPLICATION_SLOT_CLOS];
PgStat_ReplSlotStats *s = &(slotstats[i]);
MemSet(values, 0, sizeof(values));
MemSet(nulls, 0, sizeof(nulls));
values[0] = PointerGetDatum(cstring_to_text(s->slotname));
values[1] = Int64GetDatum(s->spill_txns);
values[2] = Int64GetDatum(s->spill_count);
values[3] = Int64GetDatum(s->spill_bytes);
if (s->stat_reset_timestamp == 0)
nulls[4] = true;
else
values[4] = TimestampTzGetDatum(s->stat_reset_timestamp);
tuplestore_putvalues(tupstore, tupdesc, values, nulls);
}
tuplestore_donestoring(tupstore);
return (Datum) 0;
}
...@@ -53,6 +53,6 @@ ...@@ -53,6 +53,6 @@
*/ */
/* yyyymmddN */ /* yyyymmddN */
#define CATALOG_VERSION_NO 202010021 #define CATALOG_VERSION_NO 202010081
#endif #endif
...@@ -5257,6 +5257,14 @@ ...@@ -5257,6 +5257,14 @@
proargmodes => '{o,o,o,o,o,o,o,o,o,o,o,o,o,o,o}', proargmodes => '{o,o,o,o,o,o,o,o,o,o,o,o,o,o,o}',
proargnames => '{pid,status,receive_start_lsn,receive_start_tli,written_lsn,flushed_lsn,received_tli,last_msg_send_time,last_msg_receipt_time,latest_end_lsn,latest_end_time,slot_name,sender_host,sender_port,conninfo}', proargnames => '{pid,status,receive_start_lsn,receive_start_tli,written_lsn,flushed_lsn,received_tli,last_msg_send_time,last_msg_receipt_time,latest_end_lsn,latest_end_time,slot_name,sender_host,sender_port,conninfo}',
prosrc => 'pg_stat_get_wal_receiver' }, prosrc => 'pg_stat_get_wal_receiver' },
{ oid => '8595', descr => 'statistics: information about replication slots',
proname => 'pg_stat_get_replication_slots', prorows => '10', proisstrict => 'f',
proretset => 't', provolatile => 's', proparallel => 'r',
prorettype => 'record', proargtypes => '',
proallargtypes => '{text,int8,int8,int8,timestamptz}',
proargmodes => '{o,o,o,o,o}',
proargnames => '{name,spill_txns,spill_count,spill_bytes,stats_reset}',
prosrc => 'pg_stat_get_replication_slots' },
{ oid => '6118', descr => 'statistics: information about subscription', { oid => '6118', descr => 'statistics: information about subscription',
proname => 'pg_stat_get_subscription', proisstrict => 'f', provolatile => 's', proname => 'pg_stat_get_subscription', proisstrict => 'f', provolatile => 's',
proparallel => 'r', prorettype => 'record', proargtypes => 'oid', proparallel => 'r', prorettype => 'record', proargtypes => 'oid',
...@@ -5606,6 +5614,10 @@ ...@@ -5606,6 +5614,10 @@
descr => 'statistics: reset collected statistics for a single SLRU', descr => 'statistics: reset collected statistics for a single SLRU',
proname => 'pg_stat_reset_slru', proisstrict => 'f', provolatile => 'v', proname => 'pg_stat_reset_slru', proisstrict => 'f', provolatile => 'v',
prorettype => 'void', proargtypes => 'text', prosrc => 'pg_stat_reset_slru' }, prorettype => 'void', proargtypes => 'text', prosrc => 'pg_stat_reset_slru' },
{ oid => '8596',
descr => 'statistics: reset collected statistics for a single replication slot',
proname => 'pg_stat_reset_replication_slot', proisstrict => 'f', provolatile => 'v',
prorettype => 'void', proargtypes => 'text', prosrc => 'pg_stat_reset_replication_slot' },
{ oid => '3163', descr => 'current trigger depth', { oid => '3163', descr => 'current trigger depth',
proname => 'pg_trigger_depth', provolatile => 's', proparallel => 'r', proname => 'pg_trigger_depth', provolatile => 's', proparallel => 'r',
......
...@@ -56,6 +56,7 @@ typedef enum StatMsgType ...@@ -56,6 +56,7 @@ typedef enum StatMsgType
PGSTAT_MTYPE_RESETSHAREDCOUNTER, PGSTAT_MTYPE_RESETSHAREDCOUNTER,
PGSTAT_MTYPE_RESETSINGLECOUNTER, PGSTAT_MTYPE_RESETSINGLECOUNTER,
PGSTAT_MTYPE_RESETSLRUCOUNTER, PGSTAT_MTYPE_RESETSLRUCOUNTER,
PGSTAT_MTYPE_RESETREPLSLOTCOUNTER,
PGSTAT_MTYPE_AUTOVAC_START, PGSTAT_MTYPE_AUTOVAC_START,
PGSTAT_MTYPE_VACUUM, PGSTAT_MTYPE_VACUUM,
PGSTAT_MTYPE_ANALYZE, PGSTAT_MTYPE_ANALYZE,
...@@ -68,7 +69,8 @@ typedef enum StatMsgType ...@@ -68,7 +69,8 @@ typedef enum StatMsgType
PGSTAT_MTYPE_RECOVERYCONFLICT, PGSTAT_MTYPE_RECOVERYCONFLICT,
PGSTAT_MTYPE_TEMPFILE, PGSTAT_MTYPE_TEMPFILE,
PGSTAT_MTYPE_DEADLOCK, PGSTAT_MTYPE_DEADLOCK,
PGSTAT_MTYPE_CHECKSUMFAILURE PGSTAT_MTYPE_CHECKSUMFAILURE,
PGSTAT_MTYPE_REPLSLOT,
} StatMsgType; } StatMsgType;
/* ---------- /* ----------
...@@ -358,6 +360,18 @@ typedef struct PgStat_MsgResetslrucounter ...@@ -358,6 +360,18 @@ typedef struct PgStat_MsgResetslrucounter
int m_index; int m_index;
} PgStat_MsgResetslrucounter; } PgStat_MsgResetslrucounter;
/* ----------
* PgStat_MsgResetreplslotcounter Sent by the backend to tell the collector
* to reset replication slot counter(s)
* ----------
*/
typedef struct PgStat_MsgResetreplslotcounter
{
PgStat_MsgHdr m_hdr;
char m_slotname[NAMEDATALEN];
bool clearall;
} PgStat_MsgResetreplslotcounter;
/* ---------- /* ----------
* PgStat_MsgAutovacStart Sent by the autovacuum daemon to signal * PgStat_MsgAutovacStart Sent by the autovacuum daemon to signal
* that a database is going to be processed * that a database is going to be processed
...@@ -465,6 +479,22 @@ typedef struct PgStat_MsgSLRU ...@@ -465,6 +479,22 @@ typedef struct PgStat_MsgSLRU
PgStat_Counter m_truncate; PgStat_Counter m_truncate;
} PgStat_MsgSLRU; } PgStat_MsgSLRU;
/* ----------
* PgStat_MsgReplSlot Sent by a backend or a wal sender to update replication
* slot statistics.
* ----------
*/
typedef struct PgStat_MsgReplSlot
{
PgStat_MsgHdr m_hdr;
char m_slotname[NAMEDATALEN];
bool m_drop;
PgStat_Counter m_spill_txns;
PgStat_Counter m_spill_count;
PgStat_Counter m_spill_bytes;
} PgStat_MsgReplSlot;
/* ---------- /* ----------
* PgStat_MsgRecoveryConflict Sent by the backend upon recovery conflict * PgStat_MsgRecoveryConflict Sent by the backend upon recovery conflict
* ---------- * ----------
...@@ -603,6 +633,7 @@ typedef union PgStat_Msg ...@@ -603,6 +633,7 @@ typedef union PgStat_Msg
PgStat_MsgResetsharedcounter msg_resetsharedcounter; PgStat_MsgResetsharedcounter msg_resetsharedcounter;
PgStat_MsgResetsinglecounter msg_resetsinglecounter; PgStat_MsgResetsinglecounter msg_resetsinglecounter;
PgStat_MsgResetslrucounter msg_resetslrucounter; PgStat_MsgResetslrucounter msg_resetslrucounter;
PgStat_MsgResetreplslotcounter msg_resetreplslotcounter;
PgStat_MsgAutovacStart msg_autovacuum_start; PgStat_MsgAutovacStart msg_autovacuum_start;
PgStat_MsgVacuum msg_vacuum; PgStat_MsgVacuum msg_vacuum;
PgStat_MsgAnalyze msg_analyze; PgStat_MsgAnalyze msg_analyze;
...@@ -616,6 +647,7 @@ typedef union PgStat_Msg ...@@ -616,6 +647,7 @@ typedef union PgStat_Msg
PgStat_MsgDeadlock msg_deadlock; PgStat_MsgDeadlock msg_deadlock;
PgStat_MsgTempFile msg_tempfile; PgStat_MsgTempFile msg_tempfile;
PgStat_MsgChecksumFailure msg_checksumfailure; PgStat_MsgChecksumFailure msg_checksumfailure;
PgStat_MsgReplSlot msg_replslot;
} PgStat_Msg; } PgStat_Msg;
...@@ -627,7 +659,7 @@ typedef union PgStat_Msg ...@@ -627,7 +659,7 @@ typedef union PgStat_Msg
* ------------------------------------------------------------ * ------------------------------------------------------------
*/ */
#define PGSTAT_FILE_FORMAT_ID 0x01A5BC9E #define PGSTAT_FILE_FORMAT_ID 0x01A5BC9F
/* ---------- /* ----------
* PgStat_StatDBEntry The collector's data per database * PgStat_StatDBEntry The collector's data per database
...@@ -782,6 +814,17 @@ typedef struct PgStat_SLRUStats ...@@ -782,6 +814,17 @@ typedef struct PgStat_SLRUStats
TimestampTz stat_reset_timestamp; TimestampTz stat_reset_timestamp;
} PgStat_SLRUStats; } PgStat_SLRUStats;
/*
* Replication slot statistics kept in the stats collector
*/
typedef struct PgStat_ReplSlotStats
{
char slotname[NAMEDATALEN];
PgStat_Counter spill_txns;
PgStat_Counter spill_count;
PgStat_Counter spill_bytes;
TimestampTz stat_reset_timestamp;
} PgStat_ReplSlotStats;
/* ---------- /* ----------
* Backend states * Backend states
...@@ -1330,6 +1373,7 @@ extern void pgstat_reset_counters(void); ...@@ -1330,6 +1373,7 @@ extern void pgstat_reset_counters(void);
extern void pgstat_reset_shared_counters(const char *); extern void pgstat_reset_shared_counters(const char *);
extern void pgstat_reset_single_counter(Oid objectid, PgStat_Single_Reset_Type type); extern void pgstat_reset_single_counter(Oid objectid, PgStat_Single_Reset_Type type);
extern void pgstat_reset_slru_counter(const char *); extern void pgstat_reset_slru_counter(const char *);
extern void pgstat_reset_replslot_counter(const char *name);
extern void pgstat_report_autovac(Oid dboid); extern void pgstat_report_autovac(Oid dboid);
extern void pgstat_report_vacuum(Oid tableoid, bool shared, extern void pgstat_report_vacuum(Oid tableoid, bool shared,
...@@ -1342,6 +1386,9 @@ extern void pgstat_report_recovery_conflict(int reason); ...@@ -1342,6 +1386,9 @@ extern void pgstat_report_recovery_conflict(int reason);
extern void pgstat_report_deadlock(void); extern void pgstat_report_deadlock(void);
extern void pgstat_report_checksum_failures_in_db(Oid dboid, int failurecount); extern void pgstat_report_checksum_failures_in_db(Oid dboid, int failurecount);
extern void pgstat_report_checksum_failure(void); extern void pgstat_report_checksum_failure(void);
extern void pgstat_report_replslot(const char *slotname, int spilltxns, int spillcount,
int spillbytes);
extern void pgstat_report_replslot_drop(const char *slotname);
extern void pgstat_initialize(void); extern void pgstat_initialize(void);
extern void pgstat_bestart(void); extern void pgstat_bestart(void);
...@@ -1508,6 +1555,7 @@ extern PgStat_ArchiverStats *pgstat_fetch_stat_archiver(void); ...@@ -1508,6 +1555,7 @@ extern PgStat_ArchiverStats *pgstat_fetch_stat_archiver(void);
extern PgStat_GlobalStats *pgstat_fetch_global(void); extern PgStat_GlobalStats *pgstat_fetch_global(void);
extern PgStat_WalStats *pgstat_fetch_stat_wal(void); extern PgStat_WalStats *pgstat_fetch_stat_wal(void);
extern PgStat_SLRUStats *pgstat_fetch_slru(void); extern PgStat_SLRUStats *pgstat_fetch_slru(void);
extern PgStat_ReplSlotStats *pgstat_fetch_replslot(int *nslots_p);
extern void pgstat_count_slru_page_zeroed(int slru_idx); extern void pgstat_count_slru_page_zeroed(int slru_idx);
extern void pgstat_count_slru_page_hit(int slru_idx); extern void pgstat_count_slru_page_hit(int slru_idx);
......
...@@ -122,5 +122,6 @@ extern void LogicalConfirmReceivedLocation(XLogRecPtr lsn); ...@@ -122,5 +122,6 @@ extern void LogicalConfirmReceivedLocation(XLogRecPtr lsn);
extern bool filter_by_origin_cb_wrapper(LogicalDecodingContext *ctx, RepOriginId origin_id); extern bool filter_by_origin_cb_wrapper(LogicalDecodingContext *ctx, RepOriginId origin_id);
extern void ResetLogicalStreamingState(void); extern void ResetLogicalStreamingState(void);
extern void UpdateDecodingStats(LogicalDecodingContext *ctx);
#endif #endif
...@@ -162,9 +162,10 @@ typedef struct ReorderBufferChange ...@@ -162,9 +162,10 @@ typedef struct ReorderBufferChange
#define RBTXN_HAS_CATALOG_CHANGES 0x0001 #define RBTXN_HAS_CATALOG_CHANGES 0x0001
#define RBTXN_IS_SUBXACT 0x0002 #define RBTXN_IS_SUBXACT 0x0002
#define RBTXN_IS_SERIALIZED 0x0004 #define RBTXN_IS_SERIALIZED 0x0004
#define RBTXN_IS_STREAMED 0x0008 #define RBTXN_IS_SERIALIZED_CLEAR 0x0008
#define RBTXN_HAS_TOAST_INSERT 0x0010 #define RBTXN_IS_STREAMED 0x0010
#define RBTXN_HAS_SPEC_INSERT 0x0020 #define RBTXN_HAS_TOAST_INSERT 0x0020
#define RBTXN_HAS_SPEC_INSERT 0x0040
/* Does the transaction have catalog changes? */ /* Does the transaction have catalog changes? */
#define rbtxn_has_catalog_changes(txn) \ #define rbtxn_has_catalog_changes(txn) \
...@@ -184,6 +185,12 @@ typedef struct ReorderBufferChange ...@@ -184,6 +185,12 @@ typedef struct ReorderBufferChange
((txn)->txn_flags & RBTXN_IS_SERIALIZED) != 0 \ ((txn)->txn_flags & RBTXN_IS_SERIALIZED) != 0 \
) )
/* Has this transaction ever been spilled to disk? */
#define rbtxn_is_serialized_clear(txn) \
( \
((txn)->txn_flags & RBTXN_IS_SERIALIZED_CLEAR) != 0 \
)
/* This transaction's changes has toast insert, without main table insert. */ /* This transaction's changes has toast insert, without main table insert. */
#define rbtxn_has_toast_insert(txn) \ #define rbtxn_has_toast_insert(txn) \
( \ ( \
...@@ -525,6 +532,17 @@ struct ReorderBuffer ...@@ -525,6 +532,17 @@ struct ReorderBuffer
/* memory accounting */ /* memory accounting */
Size size; Size size;
/*
* Statistics about transactions spilled to disk.
*
* A single transaction may be spilled repeatedly, which is why we keep
* two different counters. For spilling, the transaction counter includes
* both toplevel transactions and subtransactions.
*/
int64 spillTxns; /* number of transactions spilled to disk */
int64 spillCount; /* spill-to-disk invocation counter */
int64 spillBytes; /* amount of data spilled to disk */
}; };
......
...@@ -210,6 +210,7 @@ extern XLogRecPtr ReplicationSlotsComputeLogicalRestartLSN(void); ...@@ -210,6 +210,7 @@ extern XLogRecPtr ReplicationSlotsComputeLogicalRestartLSN(void);
extern bool ReplicationSlotsCountDBSlots(Oid dboid, int *nslots, int *nactive); extern bool ReplicationSlotsCountDBSlots(Oid dboid, int *nslots, int *nactive);
extern void ReplicationSlotsDropDBSlots(Oid dboid); extern void ReplicationSlotsDropDBSlots(Oid dboid);
extern void InvalidateObsoleteReplicationSlots(XLogSegNo oldestSegno); extern void InvalidateObsoleteReplicationSlots(XLogSegNo oldestSegno);
extern ReplicationSlot *SearchNamedReplicationSlot(const char *name);
extern void StartupReplicationSlots(void); extern void StartupReplicationSlots(void);
extern void CheckPointReplicationSlots(void); extern void CheckPointReplicationSlots(void);
......
...@@ -2018,6 +2018,12 @@ pg_stat_replication| SELECT s.pid, ...@@ -2018,6 +2018,12 @@ pg_stat_replication| SELECT s.pid,
FROM ((pg_stat_get_activity(NULL::integer) s(datid, pid, usesysid, application_name, state, query, wait_event_type, wait_event, xact_start, query_start, backend_start, state_change, client_addr, client_hostname, client_port, backend_xid, backend_xmin, backend_type, ssl, sslversion, sslcipher, sslbits, sslcompression, ssl_client_dn, ssl_client_serial, ssl_issuer_dn, gss_auth, gss_princ, gss_enc, leader_pid) FROM ((pg_stat_get_activity(NULL::integer) s(datid, pid, usesysid, application_name, state, query, wait_event_type, wait_event, xact_start, query_start, backend_start, state_change, client_addr, client_hostname, client_port, backend_xid, backend_xmin, backend_type, ssl, sslversion, sslcipher, sslbits, sslcompression, ssl_client_dn, ssl_client_serial, ssl_issuer_dn, gss_auth, gss_princ, gss_enc, leader_pid)
JOIN pg_stat_get_wal_senders() w(pid, state, sent_lsn, write_lsn, flush_lsn, replay_lsn, write_lag, flush_lag, replay_lag, sync_priority, sync_state, reply_time) ON ((s.pid = w.pid))) JOIN pg_stat_get_wal_senders() w(pid, state, sent_lsn, write_lsn, flush_lsn, replay_lsn, write_lag, flush_lag, replay_lag, sync_priority, sync_state, reply_time) ON ((s.pid = w.pid)))
LEFT JOIN pg_authid u ON ((s.usesysid = u.oid))); LEFT JOIN pg_authid u ON ((s.usesysid = u.oid)));
pg_stat_replication_slots| SELECT s.name,
s.spill_txns,
s.spill_count,
s.spill_bytes,
s.stats_reset
FROM pg_stat_get_replication_slots() s(name, spill_txns, spill_count, spill_bytes, stats_reset);
pg_stat_slru| SELECT s.name, pg_stat_slru| SELECT s.name,
s.blks_zeroed, s.blks_zeroed,
s.blks_hit, s.blks_hit,
......
...@@ -1832,7 +1832,9 @@ PgStat_MsgFuncstat ...@@ -1832,7 +1832,9 @@ PgStat_MsgFuncstat
PgStat_MsgHdr PgStat_MsgHdr
PgStat_MsgInquiry PgStat_MsgInquiry
PgStat_MsgRecoveryConflict PgStat_MsgRecoveryConflict
PgStat_MsgReplSlot
PgStat_MsgResetcounter PgStat_MsgResetcounter
PgStat_MsgResetreplslotcounter
PgStat_MsgResetsharedcounter PgStat_MsgResetsharedcounter
PgStat_MsgResetsinglecounter PgStat_MsgResetsinglecounter
PgStat_MsgResetslrucounter PgStat_MsgResetslrucounter
...@@ -1842,6 +1844,7 @@ PgStat_MsgTabstat ...@@ -1842,6 +1844,7 @@ PgStat_MsgTabstat
PgStat_MsgTempFile PgStat_MsgTempFile
PgStat_MsgVacuum PgStat_MsgVacuum
PgStat_MsgWal PgStat_MsgWal
PgStat_ReplSlotStats
PgStat_SLRUStats PgStat_SLRUStats
PgStat_Shared_Reset_Target PgStat_Shared_Reset_Target
PgStat_Single_Reset_Type PgStat_Single_Reset_Type
......
Markdown is supported
0% or
You are about to add 0 people to the discussion. Proceed with caution.
Finish editing this message first!
Please register or to comment