Skip to content

Commit

Permalink
Report Bloom filter size (ydb-platform#7729)
Browse files Browse the repository at this point in the history
  • Loading branch information
kunga authored Aug 16, 2024
1 parent d569f4b commit 87d8083
Show file tree
Hide file tree
Showing 16 changed files with 46 additions and 29 deletions.
2 changes: 2 additions & 0 deletions ydb/core/protos/sys_view.proto
Original file line number Diff line number Diff line change
Expand Up @@ -44,6 +44,8 @@ message TPartitionStats {
optional uint64 TxRejectedBySpace = 19;

optional TTtlStats TtlStats = 20;

optional uint64 ByKeyFilterSize = 21;
}

message TPartitionStatsResult {
Expand Down
2 changes: 2 additions & 0 deletions ydb/core/protos/table_stats.proto
Original file line number Diff line number Diff line change
Expand Up @@ -66,4 +66,6 @@ message TTableStats {
repeated TChannelStats Channels = 30;

optional TStoragePoolsStats StoragePools = 31;

optional uint64 ByKeyFilterSize = 32;
}
3 changes: 3 additions & 0 deletions ydb/core/tablet_flat/flat_stat_table.h
Original file line number Diff line number Diff line change
Expand Up @@ -102,13 +102,15 @@ struct TStats {
ui64 RowCount = 0;
TChanneledDataSize DataSize = { };
TChanneledDataSize IndexSize = { };
ui64 ByKeyFilterSize = 0;
THistogram RowCountHistogram;
THistogram DataSizeHistogram;

void Clear() {
RowCount = 0;
DataSize = { };
IndexSize = { };
ByKeyFilterSize = 0;
RowCountHistogram.clear();
DataSizeHistogram.clear();
}
Expand All @@ -117,6 +119,7 @@ struct TStats {
std::swap(RowCount, other.RowCount);
std::swap(DataSize, other.DataSize);
std::swap(IndexSize, other.IndexSize);
std::swap(ByKeyFilterSize, other.ByKeyFilterSize);
RowCountHistogram.swap(other.RowCountHistogram);
DataSizeHistogram.swap(other.DataSizeHistogram);
}
Expand Down
1 change: 1 addition & 0 deletions ydb/core/tablet_flat/flat_stat_table_btree_index.cpp
Original file line number Diff line number Diff line change
Expand Up @@ -200,6 +200,7 @@ bool BuildStatsBTreeIndex(const TSubset& subset, TStats& stats, ui32 histogramBu
bool ready = true;
for (const auto& part : subset.Flatten) {
stats.IndexSize.Add(part->IndexesRawSize, part->Label.Channel());
stats.ByKeyFilterSize += part->ByKey ? part->ByKey->Raw.size() : 0;
ready &= AddDataSize(part, stats, env, yieldHandler);
}

Expand Down
1 change: 1 addition & 0 deletions ydb/core/tablet_flat/flat_stat_table_mixed_index.h
Original file line number Diff line number Diff line change
Expand Up @@ -22,6 +22,7 @@ inline bool BuildStatsMixedIndex(const TSubset& subset, TStats& stats, ui64 rowC
bool started = true;
for (const auto& part : subset.Flatten) {
stats.IndexSize.Add(part->IndexesRawSize, part->Label.Channel());
stats.ByKeyFilterSize += part->ByKey ? part->ByKey->Raw.size() : 0;
TAutoPtr<TStatsScreenedPartIterator> iter = new TStatsScreenedPartIterator(part, env, subset.Scheme->Keys, part->Small, part->Large,
rowCountResolution / resolutionDivider, dataSizeResolution / resolutionDivider);
auto ready = iter->Start();
Expand Down
28 changes: 14 additions & 14 deletions ydb/core/tx/datashard/datashard__stats.cpp
Original file line number Diff line number Diff line change
Expand Up @@ -133,7 +133,6 @@ class TTableStatsCoroBuilder : public TActorCoroImpl, private IPages {

auto ev = MakeHolder<TDataShard::TEvPrivate::TEvAsyncTableStats>();
ev->TableId = TableId;
ev->IndexSize = IndexSize;
ev->StatsUpdateTime = StatsUpdateTime;
ev->PartCount = Subset->Flatten.size() + Subset->ColdParts.size();
ev->MemRowCount = MemRowCount;
Expand Down Expand Up @@ -272,17 +271,14 @@ class TDataShard::TTxGetTableStats : public NTabletFlatExecutor::TTransactionBas

const TUserTable& tableInfo = *Self->TableInfos[tableId];

auto indexSize = txc.DB.GetTableIndexSize(tableInfo.LocalTid);
// Fill stats with current mem table size:
auto memSize = txc.DB.GetTableMemSize(tableInfo.LocalTid);
auto memRowCount = txc.DB.GetTableMemRowCount(tableInfo.LocalTid);

if (tableInfo.ShadowTid) {
indexSize += txc.DB.GetTableIndexSize(tableInfo.ShadowTid);
memSize += txc.DB.GetTableMemSize(tableInfo.ShadowTid);
memRowCount += txc.DB.GetTableMemRowCount(tableInfo.ShadowTid);
}

Result->Record.MutableTableStats()->SetIndexSize(indexSize);
Result->Record.MutableTableStats()->SetInMemSize(memSize);
Result->Record.MutableTableStats()->SetLastAccessTime(tableInfo.Stats.AccessTime.MilliSeconds());
Result->Record.MutableTableStats()->SetLastUpdateTime(tableInfo.Stats.UpdateTime.MilliSeconds());
Expand All @@ -291,18 +287,21 @@ class TDataShard::TTxGetTableStats : public NTabletFlatExecutor::TTransactionBas
tableInfo.Stats.RowCountResolution = Ev->Get()->Record.GetRowCountResolution();
tableInfo.Stats.HistogramBucketsCount = Ev->Get()->Record.GetHistogramBucketsCount();

// Check if first stats update has been completed
// Check if first stats update has been completed:
bool ready = (tableInfo.Stats.StatsUpdateTime != TInstant());
Result->Record.SetFullStatsReady(ready);
if (!ready)
if (!ready) {
return true;
}

const TStats& stats = tableInfo.Stats.DataStats;
Result->Record.MutableTableStats()->SetIndexSize(stats.IndexSize.Size);
Result->Record.MutableTableStats()->SetByKeyFilterSize(stats.ByKeyFilterSize);
Result->Record.MutableTableStats()->SetDataSize(stats.DataSize.Size + memSize);
Result->Record.MutableTableStats()->SetRowCount(stats.RowCount + memRowCount);
FillHistogram(stats.DataSizeHistogram, *Result->Record.MutableTableStats()->MutableDataSizeHistogram());
FillHistogram(stats.RowCountHistogram, *Result->Record.MutableTableStats()->MutableRowCountHistogram());
// Fill key access sample if it was collected not too long ago
// Fill key access sample if it was collected not too long ago:
if (Self->StopKeyAccessSamplingAt + TDuration::Seconds(30) >= AppData(ctx)->TimeProvider->Now()) {
FillKeyAccessSample(tableInfo.Stats.AccessStats, *Result->Record.MutableTableStats()->MutableKeyAccessSample());
}
Expand All @@ -317,7 +316,7 @@ class TDataShard::TTxGetTableStats : public NTabletFlatExecutor::TTransactionBas
Result->Record.AddUserTablePartOwners(pi);
}

for (const auto& pi : Self->SysTablesPartOnwers) {
for (const auto& pi : Self->SysTablesPartOwners) {
Result->Record.AddSysTablesPartOwners(pi);
}

Expand Down Expand Up @@ -375,9 +374,10 @@ void TDataShard::Handle(TEvPrivate::TEvAsyncTableStats::TPtr& ev, const TActorCo
LOG_ERROR(ctx, NKikimrServices::TX_DATASHARD,
"Unexpected async stats update at datashard %" PRIu64, TabletID());
}
tableInfo.Stats.Update(std::move(ev->Get()->Stats), ev->Get()->IndexSize,
std::move(ev->Get()->PartOwners), ev->Get()->PartCount,
ev->Get()->StatsUpdateTime);
tableInfo.Stats.DataStats = std::move(ev->Get()->Stats);
tableInfo.Stats.PartOwners = std::move(ev->Get()->PartOwners);
tableInfo.Stats.PartCount = ev->Get()->PartCount;
tableInfo.Stats.StatsUpdateTime = ev->Get()->StatsUpdateTime;
tableInfo.Stats.MemRowCount = ev->Get()->MemRowCount;
tableInfo.Stats.MemDataSize = ev->Get()->MemDataSize;

Expand Down Expand Up @@ -565,12 +565,12 @@ class TDataShard::TTxInitiateStatsUpdate : public NTabletFlatExecutor::TTransact
Self->Actors.insert(actorId);
}

Self->SysTablesPartOnwers.clear();
Self->SysTablesPartOwners.clear();
for (ui32 sysTableId : Self->SysTablesToTransferAtSplit) {
THashSet<ui64> sysPartOwners;
auto subset = txc.DB.Subset(sysTableId, TEpoch::Max(), { }, { });
GetPartOwners(*subset, sysPartOwners);
Self->SysTablesPartOnwers.insert(sysPartOwners.begin(), sysPartOwners.end());
Self->SysTablesPartOwners.insert(sysPartOwners.begin(), sysPartOwners.end());
}
return true;
}
Expand Down
8 changes: 4 additions & 4 deletions ydb/core/tx/datashard/datashard_impl.h
Original file line number Diff line number Diff line change
Expand Up @@ -403,7 +403,6 @@ class TDataShard

struct TEvAsyncTableStats : public TEventLocal<TEvAsyncTableStats, EvAsyncTableStats> {
ui64 TableId = -1;
ui64 IndexSize = 0;
TInstant StatsUpdateTime;
NTable::TStats Stats;
THashSet<ui64> PartOwners;
Expand Down Expand Up @@ -2634,7 +2633,7 @@ class TDataShard
Schema::PlanQueue::TableId,
Schema::DeadlineQueue::TableId
};
THashSet<ui64> SysTablesPartOnwers;
THashSet<ui64> SysTablesPartOwners;

// Sys table contents
ui32 State;
Expand All @@ -2649,7 +2648,7 @@ class TDataShard

NMiniKQL::IKeyAccessSampler::TPtr DisabledKeySampler;
NMiniKQL::IKeyAccessSampler::TPtr EnabledKeySampler;
NMiniKQL::IKeyAccessSampler::TPtr CurrentKeySampler; // Points to enbaled or disabled
NMiniKQL::IKeyAccessSampler::TPtr CurrentKeySampler; // Points to enabled or disabled
TInstant StartedKeyAccessSamplingAt;
TInstant StopKeyAccessSamplingAt;

Expand Down Expand Up @@ -3241,6 +3240,7 @@ class TDataShard

ev->Record.MutableTableStats()->SetDataSize(ti.Stats.DataStats.DataSize.Size + ti.Stats.MemDataSize);
ev->Record.MutableTableStats()->SetIndexSize(ti.Stats.DataStats.IndexSize.Size);
ev->Record.MutableTableStats()->SetByKeyFilterSize(ti.Stats.DataStats.ByKeyFilterSize);
ev->Record.MutableTableStats()->SetInMemSize(ti.Stats.MemDataSize);

TMap<ui8, std::tuple<ui64, ui64>> channels; // Channel -> (DataSize, IndexSize)
Expand Down Expand Up @@ -3294,7 +3294,7 @@ class TDataShard
for (const auto& pi : ti.Stats.PartOwners) {
ev->Record.AddUserTablePartOwners(pi);
}
for (const auto& pi : SysTablesPartOnwers) {
for (const auto& pi : SysTablesPartOwners) {
ev->Record.AddSysTablesPartOwners(pi);
}

Expand Down
9 changes: 0 additions & 9 deletions ydb/core/tx/datashard/datashard_user_table.h
Original file line number Diff line number Diff line change
Expand Up @@ -354,7 +354,6 @@ struct TUserTable : public TThrRefBase {

struct TStats {
NTable::TStats DataStats;
ui64 IndexSize = 0;
ui64 MemRowCount = 0;
ui64 MemDataSize = 0;
TInstant AccessTime;
Expand All @@ -371,14 +370,6 @@ struct TUserTable : public TThrRefBase {
ui64 BackgroundCompactionCount = 0;
ui64 CompactBorrowedCount = 0;
NTable::TKeyAccessSample AccessStats;

void Update(NTable::TStats&& dataStats, ui64 indexSize, THashSet<ui64>&& partOwners, ui64 partCount, TInstant statsUpdateTime) {
DataStats = dataStats;
IndexSize = indexSize;
PartOwners = partOwners;
PartCount = partCount;
StatsUpdateTime = statsUpdateTime;
}
};

struct TSpecialUpdate {
Expand Down
1 change: 1 addition & 0 deletions ydb/core/tx/schemeshard/schemeshard__init.cpp
Original file line number Diff line number Diff line change
Expand Up @@ -2285,6 +2285,7 @@ struct TSchemeShard::TTxInit : public TTransactionBase<TSchemeShard> {
stats.RowCount = rowSet.GetValue<Schema::TablePartitionStats::RowCount>();
stats.DataSize = rowSet.GetValue<Schema::TablePartitionStats::DataSize>();
stats.IndexSize = rowSet.GetValue<Schema::TablePartitionStats::IndexSize>();
stats.ByKeyFilterSize = rowSet.GetValue<Schema::TablePartitionStats::ByKeyFilterSize>();
if (rowSet.HaveValue<Schema::TablePartitionStats::StoragePoolsStats>()) {
NKikimrTableStats::TStoragePoolsStats protobufRepresentation;
Y_ABORT_UNLESS(ParseFromStringNoSizeLimit(
Expand Down
2 changes: 2 additions & 0 deletions ydb/core/tx/schemeshard/schemeshard__table_stats.cpp
Original file line number Diff line number Diff line change
Expand Up @@ -64,6 +64,7 @@ auto TSchemeShard::BuildStatsForCollector(TPathId pathId, TShardIdx shardIdx, TT
sysStats.SetDataSize(stats.DataSize);
sysStats.SetRowCount(stats.RowCount);
sysStats.SetIndexSize(stats.IndexSize);
sysStats.SetByKeyFilterSize(stats.ByKeyFilterSize);
sysStats.SetCPUCores(std::min(stats.GetCurrentRawCpuUsage() / 1000000., 1.0));
sysStats.SetTabletId(ui64(datashardId));
sysStats.SetAccessTime(stats.LastAccessTime.MilliSeconds());
Expand Down Expand Up @@ -164,6 +165,7 @@ TPartitionStats TTxStoreTableStats::PrepareStats(const TActorContext& ctx,
newStats.RowCount = tableStats.GetRowCount();
newStats.DataSize = tableStats.GetDataSize();
newStats.IndexSize = tableStats.GetIndexSize();
newStats.ByKeyFilterSize = tableStats.GetByKeyFilterSize();
newStats.LastAccessTime = TInstant::MilliSeconds(tableStats.GetLastAccessTime());
newStats.LastUpdateTime = TInstant::MilliSeconds(tableStats.GetLastUpdateTime());
for (const auto& channelStats : tableStats.GetChannels()) {
Expand Down
1 change: 1 addition & 0 deletions ydb/core/tx/schemeshard/schemeshard_impl.cpp
Original file line number Diff line number Diff line change
Expand Up @@ -2506,6 +2506,7 @@ void TSchemeShard::PersistTablePartitionStats(NIceDb::TNiceDb& db, const TPathId
NIceDb::TUpdate<Schema::TablePartitionStats::RowCount>(stats.RowCount),
NIceDb::TUpdate<Schema::TablePartitionStats::DataSize>(stats.DataSize),
NIceDb::TUpdate<Schema::TablePartitionStats::IndexSize>(stats.IndexSize),
NIceDb::TUpdate<Schema::TablePartitionStats::ByKeyFilterSize>(stats.ByKeyFilterSize),

NIceDb::TUpdate<Schema::TablePartitionStats::LastAccessTime>(stats.LastAccessTime.GetValue()),
NIceDb::TUpdate<Schema::TablePartitionStats::LastUpdateTime>(stats.LastUpdateTime.GetValue()),
Expand Down
2 changes: 2 additions & 0 deletions ydb/core/tx/schemeshard/schemeshard_info_types.cpp
Original file line number Diff line number Diff line change
Expand Up @@ -1592,6 +1592,7 @@ void TTableInfo::SetPartitioning(TVector<TTableShardInfo>&& newPartitioning) {
newAggregatedStats.RowCount += newStats.RowCount;
newAggregatedStats.DataSize += newStats.DataSize;
newAggregatedStats.IndexSize += newStats.IndexSize;
newAggregatedStats.ByKeyFilterSize += newStats.ByKeyFilterSize;
for (const auto& [poolKind, newStoragePoolStats] : newStats.StoragePoolsStats) {
auto& [dataSize, indexSize] = newAggregatedStats.StoragePoolsStats[poolKind];
dataSize += newStoragePoolStats.DataSize;
Expand Down Expand Up @@ -1678,6 +1679,7 @@ void TAggregatedStats::UpdateShardStats(TShardIdx datashardIdx, const TPartition
Aggregated.RowCount += (newStats.RowCount - oldStats.RowCount);
Aggregated.DataSize += (newStats.DataSize - oldStats.DataSize);
Aggregated.IndexSize += (newStats.IndexSize - oldStats.IndexSize);
Aggregated.ByKeyFilterSize += (newStats.ByKeyFilterSize - oldStats.ByKeyFilterSize);
for (const auto& [poolKind, newStoragePoolStats] : newStats.StoragePoolsStats) {
auto& [dataSize, indexSize] = Aggregated.StoragePoolsStats[poolKind];
const auto* oldStoragePoolStats = oldStats.StoragePoolsStats.FindPtr(poolKind);
Expand Down
1 change: 1 addition & 0 deletions ydb/core/tx/schemeshard/schemeshard_info_types.h
Original file line number Diff line number Diff line change
Expand Up @@ -231,6 +231,7 @@ struct TPartitionStats {
ui64 RowCount = 0;
ui64 DataSize = 0;
ui64 IndexSize = 0;
ui64 ByKeyFilterSize = 0;

struct TStoragePoolStats {
ui64 DataSize = 0;
Expand Down
1 change: 1 addition & 0 deletions ydb/core/tx/schemeshard/schemeshard_path_describer.cpp
Original file line number Diff line number Diff line change
Expand Up @@ -25,6 +25,7 @@ static void FillTableStats(NKikimrTableStats::TTableStats* stats, const TPartiti
stats->SetRowCount(tableStats.RowCount);
stats->SetDataSize(tableStats.DataSize);
stats->SetIndexSize(tableStats.IndexSize);
stats->SetByKeyFilterSize(tableStats.ByKeyFilterSize);
stats->SetLastAccessTime(tableStats.LastAccessTime.MilliSeconds());
stats->SetLastUpdateTime(tableStats.LastUpdateTime.MilliSeconds());
stats->SetImmediateTxCompleted(tableStats.ImmediateTxCompleted);
Expand Down
5 changes: 4 additions & 1 deletion ydb/core/tx/schemeshard/schemeshard_schema.h
Original file line number Diff line number Diff line change
Expand Up @@ -378,6 +378,8 @@ struct Schema : NIceDb::Schema {
// Represented by NKikimrTableStats::TStoragePoolsStats.
struct StoragePoolsStats : Column<33, NScheme::NTypeIds::String> { using Type = TString; };

struct ByKeyFilterSize : Column<34, NScheme::NTypeIds::Uint64> {};

using TKey = TableKey<TableOwnerId, TableLocalId, PartitionId>;
using TColumns = TableColumns<
TableOwnerId,
Expand Down Expand Up @@ -412,7 +414,8 @@ struct Schema : NIceDb::Schema {
SearchHeight,
FullCompactionTs,
MemDataSize,
StoragePoolsStats
StoragePoolsStats,
ByKeyFilterSize
>;
};

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -6828,6 +6828,11 @@
"ColumnId": 33,
"ColumnName": "StoragePoolsStats",
"ColumnType": "String"
},
{
"ColumnId": 34,
"ColumnName": "ByKeyFilterSize",
"ColumnType": "Uint64"
}
],
"ColumnsDropped": [],
Expand Down Expand Up @@ -6866,7 +6871,8 @@
30,
31,
32,
33
33,
34
],
"RoomID": 0,
"Codec": 0,
Expand Down

0 comments on commit 87d8083

Please sign in to comment.