Logo ROOT  
Reference Guide
 
Loading...
Searching...
No Matches
RPageStorage.hxx
Go to the documentation of this file.
1/// \file ROOT/RPageStorage.hxx
2/// \ingroup NTuple ROOT7
3/// \author Jakob Blomer <jblomer@cern.ch>
4/// \date 2018-07-19
5/// \warning This is part of the ROOT 7 prototype! It will change without notice. It might trigger earthquakes. Feedback
6/// is welcome!
7
8/*************************************************************************
9 * Copyright (C) 1995-2019, Rene Brun and Fons Rademakers. *
10 * All rights reserved. *
11 * *
12 * For the licensing terms see $ROOTSYS/LICENSE. *
13 * For the list of contributors see $ROOTSYS/README/CREDITS. *
14 *************************************************************************/
15
16#ifndef ROOT7_RPageStorage
17#define ROOT7_RPageStorage
18
19#include <ROOT/RError.hxx>
20#include <ROOT/RCluster.hxx>
27#include <ROOT/RNTupleUtil.hxx>
28#include <ROOT/RPage.hxx>
29#include <ROOT/RPagePool.hxx>
30#include <ROOT/RSpan.hxx>
31#include <string_view>
32
33#include <atomic>
34#include <cassert>
35#include <cstddef>
36#include <deque>
37#include <functional>
38#include <memory>
39#include <mutex>
40#include <set>
41#include <shared_mutex>
42#include <unordered_map>
43#include <unordered_set>
44#include <vector>
45
46namespace ROOT {
47namespace Experimental {
48
49class RNTupleModel;
50
51namespace Internal {
52class RColumn;
53class RNTupleCompressor;
54struct RNTupleModelChangeset;
55class RPageAllocator;
56
57enum class EPageStorageType {
58 kSink,
59 kSource,
60};
61
62// clang-format off
63/**
64\class ROOT::Experimental::Internal::RPageStorage
65\ingroup NTuple
66\brief Common functionality of an ntuple storage for both reading and writing
67
68The RPageStore provides access to a storage container that keeps the bits of pages and clusters comprising
69an ntuple. Concrete implementations can use a TFile, a raw file, an object store, and so on.
70*/
71// clang-format on
73public:
74 /// The page checksum is a 64bit xxhash3
75 static constexpr std::size_t kNBytesPageChecksum = sizeof(std::uint64_t);
76
77 /// The interface of a task scheduler to schedule page (de)compression tasks
79 public:
80 virtual ~RTaskScheduler() = default;
81 /// Take a callable that represents a task
82 virtual void AddTask(const std::function<void(void)> &taskFunc) = 0;
83 /// Blocks until all scheduled tasks finished
84 virtual void Wait() = 0;
85 };
86
87 /// A sealed page contains the bytes of a page as written to storage (packed & compressed). It is used
88 /// as an input to UnsealPages() as well as to transfer pages between different storage media.
89 /// RSealedPage does _not_ own the buffer it is pointing to in order to not interfere with the memory management
90 /// of concrete page sink and page source implementations.
91 struct RSealedPage {
92 private:
93 const void *fBuffer = nullptr;
94 std::size_t fBufferSize = 0; ///< Size of the page payload and the trailing checksum (if available)
95 std::uint32_t fNElements = 0;
96 bool fHasChecksum = false; ///< If set, the last 8 bytes of the buffer are the xxhash of the rest of the buffer
97
98 public:
99 RSealedPage() = default;
100 RSealedPage(const void *buffer, std::size_t bufferSize, std::uint32_t nElements, bool hasChecksum = false)
101 : fBuffer(buffer), fBufferSize(bufferSize), fNElements(nElements), fHasChecksum(hasChecksum)
102 {
103 }
104 RSealedPage(const RSealedPage &other) = default;
105 RSealedPage &operator=(const RSealedPage &other) = default;
106 RSealedPage(RSealedPage &&other) = default;
107 RSealedPage &operator=(RSealedPage &&other) = default;
108
109 const void *GetBuffer() const { return fBuffer; }
110 void SetBuffer(const void *buffer) { fBuffer = buffer; }
111
112 std::size_t GetDataSize() const
113 {
116 }
117 std::size_t GetBufferSize() const { return fBufferSize; }
118 void SetBufferSize(std::size_t bufferSize) { fBufferSize = bufferSize; }
119
120 std::uint32_t GetNElements() const { return fNElements; }
121 void SetNElements(std::uint32_t nElements) { fNElements = nElements; }
122
123 bool GetHasChecksum() const { return fHasChecksum; }
124 void SetHasChecksum(bool hasChecksum) { fHasChecksum = hasChecksum; }
125
126 void ChecksumIfEnabled();
128 /// Returns a failure if the sealed page has no checksum
130 };
131
132 using SealedPageSequence_t = std::deque<RSealedPage>;
133 /// A range of sealed pages referring to the same column that can be used for vector commit
136 SealedPageSequence_t::const_iterator fFirst;
137 SealedPageSequence_t::const_iterator fLast;
138
139 RSealedPageGroup() = default;
140 RSealedPageGroup(DescriptorId_t d, SealedPageSequence_t::const_iterator b, SealedPageSequence_t::const_iterator e)
142 {
143 }
144 };
145
146protected:
148
149 /// For the time being, we will use the heap allocator for all sources and sinks. This may change in the future.
150 std::unique_ptr<RPageAllocator> fPageAllocator;
151
152 std::string fNTupleName;
155 {
156 if (!fTaskScheduler)
157 return;
159 }
160
161public:
162 explicit RPageStorage(std::string_view name);
163 RPageStorage(const RPageStorage &other) = delete;
164 RPageStorage &operator=(const RPageStorage &other) = delete;
165 RPageStorage(RPageStorage &&other) = default;
167 virtual ~RPageStorage();
168
169 /// Whether the concrete implementation is a sink or a source
171
174 RColumn *fColumn = nullptr;
175
176 /// Returns true for a valid column handle; fColumn and fPhysicalId should always either both
177 /// be valid or both be invalid.
178 explicit operator bool() const { return fPhysicalId != kInvalidDescriptorId && fColumn; }
179 };
180 /// The column handle identifies a column with the current open page storage
182
183 /// Register a new column. When reading, the column must exist in the ntuple on disk corresponding to the meta-data.
184 /// When writing, every column can only be attached once.
185 virtual ColumnHandle_t AddColumn(DescriptorId_t fieldId, RColumn &column) = 0;
186 /// Unregisters a column. A page source decreases the reference counter for the corresponding active column.
187 /// For a page sink, dropping columns is currently a no-op.
188 virtual void DropColumn(ColumnHandle_t columnHandle) = 0;
189 ColumnId_t GetColumnId(ColumnHandle_t columnHandle) const { return columnHandle.fPhysicalId; }
190
191 /// Returns the default metrics object. Subclasses might alternatively provide their own metrics object by
192 /// overriding this.
194
195 /// Returns the NTuple name.
196 const std::string &GetNTupleName() const { return fNTupleName; }
197
198 void SetTaskScheduler(RTaskScheduler *taskScheduler) { fTaskScheduler = taskScheduler; }
199}; // class RPageStorage
200
201// clang-format off
202/**
203\class ROOT::Experimental::Internal::RWritePageMemoryManager
204\ingroup NTuple
205\brief Helper to maintain a memory budget for the write pages of a set of columns
206
207The memory manager keeps track of the sum of bytes used by the write pages of a set of columns.
208It will flush (and shrink) large pages of other columns on the attempt to expand a page.
209*/
210// clang-format on
212private:
213 struct RColumnInfo {
214 RColumn *fColumn = nullptr;
215 std::size_t fCurrentPageSize = 0;
216 std::size_t fInitialPageSize = 0;
217
218 bool operator>(const RColumnInfo &other) const;
219 };
220
221 /// Sum of all the write page sizes (their capacity) of the columns in `fColumnsSortedByPageSize`
222 std::size_t fCurrentAllocatedBytes = 0;
223 /// Maximum allowed value for `fCurrentAllocatedBytes`, set from RNTupleWriteOptions::fPageBufferBudget
224 std::size_t fMaxAllocatedBytes = 0;
225 /// All columns that called `ReservePage()` (hence `TryUpdate()`) at least once,
226 /// sorted by their current write page size from large to small
227 std::set<RColumnInfo, std::greater<RColumnInfo>> fColumnsSortedByPageSize;
228
229 /// Flush columns in order of allocated write page size until the sum of all write page allocations
230 /// leaves space for at least targetAvailableSize bytes. Only use columns with a write page size larger
231 /// than pageSizeLimit.
232 bool TryEvict(std::size_t targetAvailableSize, std::size_t pageSizeLimit);
233
234public:
235 explicit RWritePageMemoryManager(std::size_t maxAllocatedBytes) : fMaxAllocatedBytes(maxAllocatedBytes) {}
236
237 /// Try to register the new write page size for the given column. Flush large columns to make space, if necessary.
238 /// If not enough space is available after all (sum of write pages would be larger than fMaxAllocatedBytes),
239 /// return false.
240 bool TryUpdate(RColumn &column, std::size_t newWritePageSize);
241};
242
243// clang-format off
244/**
245\class ROOT::Experimental::Internal::RPageSink
246\ingroup NTuple
247\brief Abstract interface to write data into an ntuple
248
249The page sink takes the list of columns and afterwards a series of page commits and cluster commits.
250The user is responsible to commit clusters at a consistent point, i.e. when all pages corresponding to data
251up to the given entry number are committed.
252
253An object of this class may either be a wrapper (for example a RPageSinkBuf) or a "persistent" sink,
254inheriting from RPagePersistentSink.
255*/
256// clang-format on
257class RPageSink : public RPageStorage {
258public:
259 using Callback_t = std::function<void(RPageSink &)>;
260
261 /// Cluster that was staged, but not yet logically appended to the RNTuple
263 std::uint64_t fNBytesWritten = 0;
265
266 struct RColumnInfo {
269 bool fIsSuppressed = false;
270 };
271
272 std::vector<RColumnInfo> fColumnInfos;
273 };
274
275protected:
276 std::unique_ptr<RNTupleWriteOptions> fOptions;
277
278 /// Helper to zip pages and header/footer; includes a 16MB (kMAXZIPBUF) zip buffer.
279 /// There could be concrete page sinks that don't need a compressor. Therefore, and in order to stay consistent
280 /// with the page source, we leave it up to the derived class whether or not the compressor gets constructed.
281 std::unique_ptr<RNTupleCompressor> fCompressor;
282
283 /// Helper for streaming a page. This is commonly used in derived, concrete page sinks. Note that if
284 /// compressionSetting is 0 (uncompressed) and the page is mappable and not checksummed, the returned sealed page
285 /// will point directly to the input page buffer. Otherwise, the sealed page references an internal buffer
286 /// of fCompressor. Thus, the buffer pointed to by the RSealedPage should never be freed.
287 /// Usage of this method requires construction of fCompressor.
288 RSealedPage SealPage(const RPage &page, const RColumnElementBase &element);
289
290private:
291 /// Flag if sink was initialized
292 bool fIsInitialized = false;
293 std::vector<Callback_t> fOnDatasetCommitCallbacks;
294 std::vector<unsigned char> fSealPageBuffer; ///< Used as destination buffer in the simple SealPage overload
295
296 /// Used in ReservePage to maintain the page buffer budget
298
299public:
300 RPageSink(std::string_view ntupleName, const RNTupleWriteOptions &options);
301
302 RPageSink(const RPageSink &) = delete;
303 RPageSink &operator=(const RPageSink &) = delete;
304 RPageSink(RPageSink &&) = default;
306 ~RPageSink() override;
307
309 /// Returns the sink's write options.
310 const RNTupleWriteOptions &GetWriteOptions() const { return *fOptions; }
311
312 void DropColumn(ColumnHandle_t /*columnHandle*/) final {}
313
314 bool IsInitialized() const { return fIsInitialized; }
315
316 /// Return the RNTupleDescriptor being constructed.
317 virtual const RNTupleDescriptor &GetDescriptor() const = 0;
318
319 /// Physically creates the storage container to hold the ntuple (e.g., a keys a TFile or an S3 bucket)
320 /// Init() associates column handles to the columns referenced by the model
321 void Init(RNTupleModel &model)
322 {
323 if (fIsInitialized) {
324 throw RException(R__FAIL("already initialized"));
325 }
326 fIsInitialized = true;
327 InitImpl(model);
328 }
329
330protected:
331 virtual void InitImpl(RNTupleModel &model) = 0;
332 virtual void CommitDatasetImpl() = 0;
333
334public:
335 /// Parameters for the SealPage() method
337 const RPage *fPage = nullptr; ///< Input page to be sealed
338 const RColumnElementBase *fElement = nullptr; ///< Corresponds to the page's elements, for size calculation etc.
339 int fCompressionSetting = 0; ///< Compression algorithm and level to apply
340 /// Adds a 8 byte little-endian xxhash3 checksum to the page payload. The buffer has to be large enough to
341 /// to store the additional 8 bytes.
342 bool fWriteChecksum = true;
343 /// If false, the output buffer must not point to the input page buffer, which would otherwise be an option
344 /// if the page is mappable and should not be compressed
345 bool fAllowAlias = false;
346 /// Location for sealed output. The memory buffer has to be large enough.
347 void *fBuffer = nullptr;
348 };
349
350 /// Seal a page using the provided info.
351 static RSealedPage SealPage(const RSealPageConfig &config);
352
353 /// Incorporate incremental changes to the model into the ntuple descriptor. This happens, e.g. if new fields were
354 /// added after the initial call to `RPageSink::Init(RNTupleModel &)`.
355 /// `firstEntry` specifies the global index for the first stored element in the added columns.
356 virtual void UpdateSchema(const RNTupleModelChangeset &changeset, NTupleSize_t firstEntry) = 0;
357 /// Adds an extra type information record to schema. The extra type information will be written to the
358 /// extension header. The information in the record will be merged with the existing information, e.g.
359 /// duplicate streamer info records will be removed. This method is called by the "on commit dataset" callback
360 /// registered by specific fields (e.g., streamer field) and during merging.
361 virtual void UpdateExtraTypeInfo(const RExtraTypeInfoDescriptor &extraTypeInfo) = 0;
362
363 /// Commits a suppressed column for the current cluster. Can be called anytime before CommitCluster().
364 /// For any given column and cluster, there must be no calls to both CommitSuppressedColumn() and page commits.
365 virtual void CommitSuppressedColumn(ColumnHandle_t columnHandle) = 0;
366 /// Write a page to the storage. The column must have been added before.
367 virtual void CommitPage(ColumnHandle_t columnHandle, const RPage &page) = 0;
368 /// Write a preprocessed page to storage. The column must have been added before.
369 virtual void CommitSealedPage(DescriptorId_t physicalColumnId, const RPageStorage::RSealedPage &sealedPage) = 0;
370 /// Write a vector of preprocessed pages to storage. The corresponding columns must have been added before.
371 virtual void CommitSealedPageV(std::span<RPageStorage::RSealedPageGroup> ranges) = 0;
372 /// Stage the current cluster and create a new one for the following data.
373 /// Returns the object that must be passed to CommitStagedClusters to logically append the staged cluster to the
374 /// ntuple descriptor.
375 virtual RStagedCluster StageCluster(NTupleSize_t nNewEntries) = 0;
376 /// Commit staged clusters, logically appending them to the ntuple descriptor.
377 virtual void CommitStagedClusters(std::span<RStagedCluster> clusters) = 0;
378 /// Finalize the current cluster and create a new one for the following data.
379 /// Returns the number of bytes written to storage (excluding meta-data).
380 virtual std::uint64_t CommitCluster(NTupleSize_t nNewEntries)
381 {
382 RStagedCluster stagedClusters[] = {StageCluster(nNewEntries)};
383 CommitStagedClusters(stagedClusters);
384 return stagedClusters[0].fNBytesWritten;
385 }
386 /// Write out the page locations (page list envelope) for all the committed clusters since the last call of
387 /// CommitClusterGroup (or the beginning of writing).
388 virtual void CommitClusterGroup() = 0;
389
390 /// The registered callback is executed at the beginning of CommitDataset();
392 /// Run the registered callbacks and finalize the current cluster and the entrire data set.
393 void CommitDataset();
394
395 /// Get a new, empty page for the given column that can be filled with up to nElements;
396 /// nElements must be larger than zero.
397 virtual RPage ReservePage(ColumnHandle_t columnHandle, std::size_t nElements);
398
399 /// An RAII wrapper used to synchronize a page sink. See GetSinkGuard().
401 std::mutex *fLock;
402
403 public:
404 explicit RSinkGuard(std::mutex *lock) : fLock(lock)
405 {
406 if (fLock != nullptr) {
407 fLock->lock();
408 }
409 }
410 RSinkGuard(const RSinkGuard &) = delete;
411 RSinkGuard &operator=(const RSinkGuard &) = delete;
412 RSinkGuard(RSinkGuard &&) = delete;
415 {
416 if (fLock != nullptr) {
417 fLock->unlock();
418 }
419 }
420 };
421
423 {
424 // By default, there is no lock and the guard does nothing.
425 return RSinkGuard(nullptr);
426 }
427}; // class RPageSink
428
429// clang-format off
430/**
431\class ROOT::Experimental::Internal::RPagePersistentSink
432\ingroup NTuple
433\brief Base class for a sink with a physical storage backend
434*/
435// clang-format on
437private:
438 /// Used to map the IDs of the descriptor to the physical IDs issued during header/footer serialization
440
441 /// Remembers the starting cluster id for the next cluster group
442 std::uint64_t fNextClusterInGroup = 0;
443 /// Used to calculate the number of entries in the current cluster
445 /// Keeps track of the number of elements in the currently open cluster. Indexed by column id.
446 std::vector<RClusterDescriptor::RColumnRange> fOpenColumnRanges;
447 /// Keeps track of the written pages in the currently open cluster. Indexed by column id.
448 std::vector<RClusterDescriptor::RPageRange> fOpenPageRanges;
449
450 /// Union of the streamer info records that are sent from streamer fields to the sink before committing the dataset.
452
453protected:
454 /// Set of optional features supported by the persistent sink
455 struct RFeatures {
456 bool fCanMergePages = false;
457 };
458
461
462 /// Default I/O performance counters that get registered in fMetrics
463 struct RCounters {
471 };
472 std::unique_ptr<RCounters> fCounters;
473
474 virtual void InitImpl(unsigned char *serializedHeader, std::uint32_t length) = 0;
475
476 virtual RNTupleLocator CommitPageImpl(ColumnHandle_t columnHandle, const RPage &page) = 0;
477 virtual RNTupleLocator
478 CommitSealedPageImpl(DescriptorId_t physicalColumnId, const RPageStorage::RSealedPage &sealedPage) = 0;
479 /// Vector commit of preprocessed pages. The `ranges` array specifies a range of sealed pages to be
480 /// committed for each column. The returned vector contains, in order, the RNTupleLocator for each
481 /// page on each range in `ranges`, i.e. the first N entries refer to the N pages in `ranges[0]`,
482 /// followed by M entries that refer to the M pages in `ranges[1]`, etc.
483 /// The mask allows to skip writing out certain pages. The vector has the size of all the pages.
484 /// For every `false` value in the mask, the corresponding locator is skipped (missing) in the output vector.
485 /// The default is to call `CommitSealedPageImpl` for each page; derived classes may provide an
486 /// optimized implementation though.
487 virtual std::vector<RNTupleLocator>
488 CommitSealedPageVImpl(std::span<RPageStorage::RSealedPageGroup> ranges, const std::vector<bool> &mask);
489 /// Returns the number of bytes written to storage (excluding metadata)
490 virtual std::uint64_t StageClusterImpl() = 0;
491 /// Returns the locator of the page list envelope of the given buffer that contains the serialized page list.
492 /// Typically, the implementation takes care of compressing and writing the provided buffer.
493 virtual RNTupleLocator CommitClusterGroupImpl(unsigned char *serializedPageList, std::uint32_t length) = 0;
494 virtual void CommitDatasetImpl(unsigned char *serializedFooter, std::uint32_t length) = 0;
495
496 /// Enables the default set of metrics provided by RPageSink. `prefix` will be used as the prefix for
497 /// the counters registered in the internal RNTupleMetrics object.
498 /// This set of counters can be extended by a subclass by calling `fMetrics.MakeCounter<...>()`.
499 ///
500 /// A subclass using the default set of metrics is always responsible for updating the counters
501 /// appropriately, e.g. `fCounters->fNPageCommited.Inc()`
502 void EnableDefaultMetrics(const std::string &prefix);
503
504public:
505 RPagePersistentSink(std::string_view ntupleName, const RNTupleWriteOptions &options);
506
511 ~RPagePersistentSink() override;
512
513 /// Guess the concrete derived page source from the location
514 static std::unique_ptr<RPageSink> Create(std::string_view ntupleName, std::string_view location,
515 const RNTupleWriteOptions &options = RNTupleWriteOptions());
516
517 ColumnHandle_t AddColumn(DescriptorId_t fieldId, RColumn &column) final;
518
520
521 /// Updates the descriptor and calls InitImpl() that handles the backend-specific details (file, DAOS, etc.)
522 void InitImpl(RNTupleModel &model) final;
523 void UpdateSchema(const RNTupleModelChangeset &changeset, NTupleSize_t firstEntry) final;
524 void UpdateExtraTypeInfo(const RExtraTypeInfoDescriptor &extraTypeInfo) final;
525
526 /// Initialize sink based on an existing descriptor and fill into the descriptor builder.
527 void InitFromDescriptor(const RNTupleDescriptor &descriptor);
528
529 void CommitSuppressedColumn(ColumnHandle_t columnHandle) final;
530 void CommitPage(ColumnHandle_t columnHandle, const RPage &page) final;
531 void CommitSealedPage(DescriptorId_t physicalColumnId, const RPageStorage::RSealedPage &sealedPage) final;
532 void CommitSealedPageV(std::span<RPageStorage::RSealedPageGroup> ranges) final;
533 RStagedCluster StageCluster(NTupleSize_t nNewEntries) final;
534 void CommitStagedClusters(std::span<RStagedCluster> clusters) final;
535 void CommitClusterGroup() final;
536 void CommitDatasetImpl() final;
537}; // class RPagePersistentSink
538
539// clang-format off
540/**
541\class ROOT::Experimental::Internal::RPageSource
542\ingroup NTuple
543\brief Abstract interface to read data from an ntuple
544
545The page source is initialized with the columns of interest. Alias columns from projected fields are mapped to the
546corresponding physical columns. Pages from the columns of interest can then be mapped into memory.
547The page source also gives access to the ntuple's meta-data.
548*/
549// clang-format on
550class RPageSource : public RPageStorage {
551public:
552 /// Used in SetEntryRange / GetEntryRange
553 struct REntryRange {
555 NTupleSize_t fNEntries = 0;
556
557 /// Returns true if the given cluster has entries within the entry range
558 bool IntersectsWith(const RClusterDescriptor &clusterDesc) const;
559 };
560
561 /// An RAII wrapper used for the read-only access to `RPageSource::fDescriptor`. See `GetExclDescriptorGuard()``.
564 std::shared_mutex &fLock;
565
566 public:
567 RSharedDescriptorGuard(const RNTupleDescriptor &desc, std::shared_mutex &lock) : fDescriptor(desc), fLock(lock)
568 {
569 fLock.lock_shared();
570 }
575 ~RSharedDescriptorGuard() { fLock.unlock_shared(); }
576 const RNTupleDescriptor *operator->() const { return &fDescriptor; }
577 const RNTupleDescriptor &GetRef() const { return fDescriptor; }
578 };
579
580 /// An RAII wrapper used for the writable access to `RPageSource::fDescriptor`. See `GetSharedDescriptorGuard()`.
583 std::shared_mutex &fLock;
584
585 public:
586 RExclDescriptorGuard(RNTupleDescriptor &desc, std::shared_mutex &lock) : fDescriptor(desc), fLock(lock)
587 {
588 fLock.lock();
589 }
595 {
596 fDescriptor.IncGeneration();
597 fLock.unlock();
598 }
599 RNTupleDescriptor *operator->() const { return &fDescriptor; }
600 void MoveIn(RNTupleDescriptor &&desc) { fDescriptor = std::move(desc); }
601 };
602
603private:
605 mutable std::shared_mutex fDescriptorLock;
606 REntryRange fEntryRange; ///< Used by the cluster pool to prevent reading beyond the given range
607 bool fHasStructure = false; ///< Set to true once `LoadStructure()` is called
608 bool fIsAttached = false; ///< Set to true once `Attach()` is called
609
610protected:
611 /// Default I/O performance counters that get registered in `fMetrics`
612 struct RCounters {
630 };
631
632 /// Keeps track of the requested physical column IDs and their in-memory target type via a column element identifier.
633 /// When using alias columns (projected fields), physical columns may be requested multiple times.
635 public:
636 struct RColumnInfo {
638 std::size_t fRefCounter = 0;
639 };
640
641 private:
642 /// Maps physical column IDs to all the requested in-memory representations.
643 /// A pair of physical column ID and in-memory representation can be requested multiple times, which is
644 /// indicated by the reference counter.
645 /// We can only have a handful of possible in-memory representations for a given column,
646 /// so it is fine to search them linearly.
647 std::unordered_map<DescriptorId_t, std::vector<RColumnInfo>> fColumnInfos;
648
649 public:
650 void Insert(DescriptorId_t physicalColumnId, RColumnElementBase::RIdentifier elementId);
651 void Erase(DescriptorId_t physicalColumnId, RColumnElementBase::RIdentifier elementId);
652 RCluster::ColumnSet_t ToColumnSet() const;
653 bool HasColumnInfos(DescriptorId_t physicalColumnId) const { return fColumnInfos.count(physicalColumnId) > 0; }
654 const std::vector<RColumnInfo> &GetColumnInfos(DescriptorId_t physicalColumnId) const
655 {
656 return fColumnInfos.at(physicalColumnId);
657 }
658 };
659
660 /// Summarizes cluster-level information that are necessary to load a certain page.
661 /// Used by LoadPageImpl().
663 DescriptorId_t fClusterId = 0;
664 /// Location of the page on disk
666 /// The first element number of the page's column in the given cluster
667 std::uint64_t fColumnOffset = 0;
668 };
669
670 std::unique_ptr<RCounters> fCounters;
671
673 /// The active columns are implicitly defined by the model fields or views
675
676 /// Pages that are unzipped with IMT are staged into the page pool
678
679 virtual void LoadStructureImpl() = 0;
680 /// `LoadStructureImpl()` has been called before `AttachImpl()` is called
682 /// Returns a new, unattached page source for the same data set
683 virtual std::unique_ptr<RPageSource> CloneImpl() const = 0;
684 // Only called if a task scheduler is set. No-op be default.
685 virtual void UnzipClusterImpl(RCluster *cluster);
686 // Returns a page from storage if not found in the page pool. Should be able to handle zero page locators.
687 virtual RPageRef LoadPageImpl(ColumnHandle_t columnHandle, const RClusterInfo &clusterInfo,
688 ClusterSize_t::ValueType idxInCluster) = 0;
689
690 /// Prepare a page range read for the column set in `clusterKey`. Specifically, pages referencing the
691 /// `kTypePageZero` locator are filled in `pageZeroMap`; otherwise, `perPageFunc` is called for each page. This is
692 /// commonly used as part of `LoadClusters()` in derived classes.
693 void PrepareLoadCluster(
694 const RCluster::RKey &clusterKey, ROnDiskPageMap &pageZeroMap,
695 std::function<void(DescriptorId_t, NTupleSize_t, const RClusterDescriptor::RPageRange::RPageInfo &)> perPageFunc);
696
697 /// Enables the default set of metrics provided by RPageSource. `prefix` will be used as the prefix for
698 /// the counters registered in the internal RNTupleMetrics object.
699 /// A subclass using the default set of metrics is responsible for updating the counters
700 /// appropriately, e.g. `fCounters->fNRead.Inc()`
701 /// Alternatively, a subclass might provide its own RNTupleMetrics object by overriding the
702 /// `GetMetrics()` member function.
703 void EnableDefaultMetrics(const std::string &prefix);
704
705 /// Note that the underlying lock is not recursive. See `GetSharedDescriptorGuard()` for further information.
706 RExclDescriptorGuard GetExclDescriptorGuard() { return RExclDescriptorGuard(fDescriptor, fDescriptorLock); }
707
708public:
709 RPageSource(std::string_view ntupleName, const RNTupleReadOptions &fOptions);
710 RPageSource(const RPageSource &) = delete;
714 ~RPageSource() override;
715 /// Guess the concrete derived page source from the file name (location)
716 static std::unique_ptr<RPageSource> Create(std::string_view ntupleName, std::string_view location,
717 const RNTupleReadOptions &options = RNTupleReadOptions());
718 /// Open the same storage multiple time, e.g. for reading in multiple threads.
719 /// If the source is already attached, the clone will be attached, too. The clone will use, however,
720 /// it's own connection to the underlying storage (e.g., file descriptor, XRootD handle, etc.)
721 std::unique_ptr<RPageSource> Clone() const;
722
723 /// Helper for unstreaming a page. This is commonly used in derived, concrete page sources. The implementation
724 /// currently always makes a memory copy, even if the sealed page is uncompressed and in the final memory layout.
725 /// The optimization of directly mapping pages is left to the concrete page source implementations.
726 RResult<RPage> static UnsealPage(const RSealedPage &sealedPage, const RColumnElementBase &element,
727 DescriptorId_t physicalColumnId, RPageAllocator &pageAlloc);
728
730 const RNTupleReadOptions &GetReadOptions() const { return fOptions; }
731
732 /// Takes the read lock for the descriptor. Multiple threads can take the lock concurrently.
733 /// The underlying `std::shared_mutex`, however, is neither read nor write recursive:
734 /// within one thread, only one lock (shared or exclusive) must be acquired at the same time. This requires special
735 /// care in sections protected by `GetSharedDescriptorGuard()` and `GetExclDescriptorGuard()` especially to avoid
736 /// that the locks are acquired indirectly (e.g. by a call to `GetNEntries()`). As a general guideline, no other
737 /// method of the page source should be called (directly or indirectly) in a guarded section.
739 {
740 return RSharedDescriptorGuard(fDescriptor, fDescriptorLock);
741 }
742
743 ColumnHandle_t AddColumn(DescriptorId_t fieldId, RColumn &column) override;
744 void DropColumn(ColumnHandle_t columnHandle) override;
745
746 /// Loads header and footer without decompressing or deserializing them. This can be used to asynchronously open
747 /// a file in the background. The method is idempotent and it is called as a first step in `Attach()`.
748 /// Pages sources may or may not make use of splitting loading and processing meta-data.
749 /// Therefore, `LoadStructure()` may do nothing and defer loading the meta-data to `Attach()`.
750 void LoadStructure();
751 /// Open the physical storage container and deserialize header and footer
752 void Attach();
753 NTupleSize_t GetNEntries();
754 NTupleSize_t GetNElements(ColumnHandle_t columnHandle);
755
756 /// Promise to only read from the given entry range. If set, prevents the cluster pool from reading-ahead beyond
757 /// the given range. The range needs to be within `[0, GetNEntries())`.
758 void SetEntryRange(const REntryRange &range);
759 REntryRange GetEntryRange() const { return fEntryRange; }
760
761 /// Allocates and fills a page that contains the index-th element. The default implementation searches
762 /// the page and calls LoadPageImpl(). Returns a default-constructed RPage for suppressed columns.
763 virtual RPageRef LoadPage(ColumnHandle_t columnHandle, NTupleSize_t globalIndex);
764 /// Another version of `LoadPage` that allows to specify cluster-relative indexes.
765 /// Returns a default-constructed RPage for suppressed columns.
766 virtual RPageRef LoadPage(ColumnHandle_t columnHandle, RClusterIndex clusterIndex);
767
768 /// Read the packed and compressed bytes of a page into the memory buffer provided by `sealedPage`. The sealed page
769 /// can be used subsequently in a call to `RPageSink::CommitSealedPage`.
770 /// The `fSize` and `fNElements` member of the sealedPage parameters are always set. If `sealedPage.fBuffer` is
771 /// `nullptr`, no data will be copied but the returned size information can be used by the caller to allocate a large
772 /// enough buffer and call `LoadSealedPage` again.
773 virtual void
774 LoadSealedPage(DescriptorId_t physicalColumnId, RClusterIndex clusterIndex, RSealedPage &sealedPage) = 0;
775
776 /// Populates all the pages of the given cluster ids and columns; it is possible that some columns do not
777 /// contain any pages. The page source may load more columns than the minimal necessary set from `columns`.
778 /// To indicate which columns have been loaded, `LoadClusters()`` must mark them with `SetColumnAvailable()`.
779 /// That includes the ones from the `columns` that don't have pages; otherwise subsequent requests
780 /// for the cluster would assume an incomplete cluster and trigger loading again.
781 /// `LoadClusters()` is typically called from the I/O thread of a cluster pool, i.e. the method runs
782 /// concurrently to other methods of the page source.
783 virtual std::vector<std::unique_ptr<RCluster>> LoadClusters(std::span<RCluster::RKey> clusterKeys) = 0;
784
785 /// Parallel decompression and unpacking of the pages in the given cluster. The unzipped pages are supposed
786 /// to be preloaded in a page pool attached to the source. The method is triggered by the cluster pool's
787 /// unzip thread. It is an optional optimization, the method can safely do nothing. In particular, the
788 /// actual implementation will only run if a task scheduler is set. In practice, a task scheduler is set
789 /// if implicit multi-threading is turned on.
790 void UnzipCluster(RCluster *cluster);
791
792 // TODO(gparolini): for symmetry with SealPage(), we should either make this private or SealPage() public.
794 UnsealPage(const RSealedPage &sealedPage, const RColumnElementBase &element, DescriptorId_t physicalColumnId);
795
796}; // class RPageSource
797
798} // namespace Internal
799
800} // namespace Experimental
801} // namespace ROOT
802
803#endif
#define R__FAIL(msg)
Short-hand to return an RResult<T> in an error state; the RError is implicitly converted into RResult...
Definition RError.hxx:290
#define d(i)
Definition RSha256.hxx:102
#define b(i)
Definition RSha256.hxx:100
#define e(i)
Definition RSha256.hxx:103
Option_t Option_t TPoint TPoint const char GetTextMagnitude GetFillStyle GetLineColor GetLineWidth GetMarkerStyle GetTextAlign GetTextColor GetTextSize void char Point_t Rectangle_t WindowAttributes_t Float_t Float_t Float_t Int_t Int_t UInt_t UInt_t Rectangle_t mask
Option_t Option_t TPoint TPoint const char GetTextMagnitude GetFillStyle GetLineColor GetLineWidth GetMarkerStyle GetTextAlign GetTextColor GetTextSize void char Point_t Rectangle_t WindowAttributes_t Float_t Float_t Float_t Int_t Int_t UInt_t UInt_t Rectangle_t Int_t Int_t Window_t TString Int_t GCValues_t GetPrimarySelectionOwner GetDisplay GetScreen GetColormap GetNativeEvent const char const char dpyName wid window const char font_name cursor keysym reg const char only_if_exist regb h Point_t winding char text const char depth char const char Int_t count const char ColorStruct_t color const char Pixmap_t Pixmap_t PictureAttributes_t attr const char char ret_data h unsigned char height h length
char name[80]
Definition TGX11.cxx:110
A thread-safe integral performance counter.
A metric element that computes its floating point value from other counters.
A collection of Counter objects with a name, a unit, and a description.
An either thread-safe or non thread safe counter for CPU ticks.
An in-memory subset of the packed and compressed pages of a cluster.
Definition RCluster.hxx:152
std::unordered_set< DescriptorId_t > ColumnSet_t
Definition RCluster.hxx:154
A column element encapsulates the translation between basic C++ types and their column representation...
A column is a storage-backed array of a simple, fixed-size type, from which pages can be mapped into ...
Definition RColumn.hxx:40
A helper class for piece-wise construction of an RNTupleDescriptor.
The serialization context is used for the piecewise serialization of a descriptor.
std::map< Int_t, TVirtualStreamerInfo * > StreamerInfoMap_t
A memory region that contains packed and compressed pages.
Definition RCluster.hxx:103
Abstract interface to allocate and release pages.
Base class for a sink with a physical storage backend.
RPagePersistentSink(const RPagePersistentSink &)=delete
ColumnHandle_t AddColumn(DescriptorId_t fieldId, RColumn &column) final
Register a new column.
RPagePersistentSink(RPagePersistentSink &&)=default
RStagedCluster StageCluster(NTupleSize_t nNewEntries) final
Stage the current cluster and create a new one for the following data.
std::uint64_t fNextClusterInGroup
Remembers the starting cluster id for the next cluster group.
virtual std::uint64_t StageClusterImpl()=0
Returns the number of bytes written to storage (excluding metadata)
RPagePersistentSink & operator=(RPagePersistentSink &&)=default
virtual void InitImpl(unsigned char *serializedHeader, std::uint32_t length)=0
virtual std::vector< RNTupleLocator > CommitSealedPageVImpl(std::span< RPageStorage::RSealedPageGroup > ranges, const std::vector< bool > &mask)
Vector commit of preprocessed pages.
RNTupleSerializer::RContext fSerializationContext
Used to map the IDs of the descriptor to the physical IDs issued during header/footer serialization.
virtual RNTupleLocator CommitPageImpl(ColumnHandle_t columnHandle, const RPage &page)=0
void InitFromDescriptor(const RNTupleDescriptor &descriptor)
Initialize sink based on an existing descriptor and fill into the descriptor builder.
virtual RNTupleLocator CommitClusterGroupImpl(unsigned char *serializedPageList, std::uint32_t length)=0
Returns the locator of the page list envelope of the given buffer that contains the serialized page l...
NTupleSize_t fPrevClusterNEntries
Used to calculate the number of entries in the current cluster.
std::vector< RClusterDescriptor::RPageRange > fOpenPageRanges
Keeps track of the written pages in the currently open cluster. Indexed by column id.
const RNTupleDescriptor & GetDescriptor() const final
Return the RNTupleDescriptor being constructed.
void CommitPage(ColumnHandle_t columnHandle, const RPage &page) final
Write a page to the storage. The column must have been added before.
virtual void CommitDatasetImpl(unsigned char *serializedFooter, std::uint32_t length)=0
RPagePersistentSink & operator=(const RPagePersistentSink &)=delete
static std::unique_ptr< RPageSink > Create(std::string_view ntupleName, std::string_view location, const RNTupleWriteOptions &options=RNTupleWriteOptions())
Guess the concrete derived page source from the location.
Internal::RNTupleDescriptorBuilder fDescriptorBuilder
RNTupleSerializer::StreamerInfoMap_t fStreamerInfos
Union of the streamer info records that are sent from streamer fields to the sink before committing t...
void CommitClusterGroup() final
Write out the page locations (page list envelope) for all the committed clusters since the last call ...
std::vector< RClusterDescriptor::RColumnRange > fOpenColumnRanges
Keeps track of the number of elements in the currently open cluster. Indexed by column id.
void UpdateSchema(const RNTupleModelChangeset &changeset, NTupleSize_t firstEntry) final
Incorporate incremental changes to the model into the ntuple descriptor.
void CommitSealedPage(DescriptorId_t physicalColumnId, const RPageStorage::RSealedPage &sealedPage) final
Write a preprocessed page to storage. The column must have been added before.
void CommitSealedPageV(std::span< RPageStorage::RSealedPageGroup > ranges) final
Write a vector of preprocessed pages to storage. The corresponding columns must have been added befor...
void CommitSuppressedColumn(ColumnHandle_t columnHandle) final
Commits a suppressed column for the current cluster.
void CommitStagedClusters(std::span< RStagedCluster > clusters) final
Commit staged clusters, logically appending them to the ntuple descriptor.
void EnableDefaultMetrics(const std::string &prefix)
Enables the default set of metrics provided by RPageSink.
void UpdateExtraTypeInfo(const RExtraTypeInfoDescriptor &extraTypeInfo) final
Adds an extra type information record to schema.
virtual RNTupleLocator CommitSealedPageImpl(DescriptorId_t physicalColumnId, const RPageStorage::RSealedPage &sealedPage)=0
A thread-safe cache of pages loaded from the page source.
Definition RPagePool.hxx:45
Reference to a page stored in the page pool.
Definition RPagePool.hxx:93
An RAII wrapper used to synchronize a page sink. See GetSinkGuard().
RSinkGuard & operator=(const RSinkGuard &)=delete
RSinkGuard & operator=(RSinkGuard &&)=delete
Abstract interface to write data into an ntuple.
std::vector< unsigned char > fSealPageBuffer
Used as destination buffer in the simple SealPage overload.
std::vector< Callback_t > fOnDatasetCommitCallbacks
RPageSink & operator=(RPageSink &&)=default
virtual RStagedCluster StageCluster(NTupleSize_t nNewEntries)=0
Stage the current cluster and create a new one for the following data.
bool fIsInitialized
Flag if sink was initialized.
virtual void UpdateExtraTypeInfo(const RExtraTypeInfoDescriptor &extraTypeInfo)=0
Adds an extra type information record to schema.
void CommitDataset()
Run the registered callbacks and finalize the current cluster and the entrire data set.
virtual void CommitSuppressedColumn(ColumnHandle_t columnHandle)=0
Commits a suppressed column for the current cluster.
virtual const RNTupleDescriptor & GetDescriptor() const =0
Return the RNTupleDescriptor being constructed.
void Init(RNTupleModel &model)
Physically creates the storage container to hold the ntuple (e.g., a keys a TFile or an S3 bucket) In...
virtual RPage ReservePage(ColumnHandle_t columnHandle, std::size_t nElements)
Get a new, empty page for the given column that can be filled with up to nElements; nElements must be...
RWritePageMemoryManager fWritePageMemoryManager
Used in ReservePage to maintain the page buffer budget.
virtual void CommitPage(ColumnHandle_t columnHandle, const RPage &page)=0
Write a page to the storage. The column must have been added before.
const RNTupleWriteOptions & GetWriteOptions() const
Returns the sink's write options.
RPageSink & operator=(const RPageSink &)=delete
virtual void CommitClusterGroup()=0
Write out the page locations (page list envelope) for all the committed clusters since the last call ...
virtual std::uint64_t CommitCluster(NTupleSize_t nNewEntries)
Finalize the current cluster and create a new one for the following data.
void RegisterOnCommitDatasetCallback(Callback_t callback)
The registered callback is executed at the beginning of CommitDataset();.
RPageSink(const RPageSink &)=delete
std::function< void(RPageSink &)> Callback_t
virtual void CommitSealedPage(DescriptorId_t physicalColumnId, const RPageStorage::RSealedPage &sealedPage)=0
Write a preprocessed page to storage. The column must have been added before.
virtual void CommitStagedClusters(std::span< RStagedCluster > clusters)=0
Commit staged clusters, logically appending them to the ntuple descriptor.
virtual void InitImpl(RNTupleModel &model)=0
void DropColumn(ColumnHandle_t) final
Unregisters a column.
EPageStorageType GetType() final
Whether the concrete implementation is a sink or a source.
std::unique_ptr< RNTupleCompressor > fCompressor
Helper to zip pages and header/footer; includes a 16MB (kMAXZIPBUF) zip buffer.
virtual void UpdateSchema(const RNTupleModelChangeset &changeset, NTupleSize_t firstEntry)=0
Incorporate incremental changes to the model into the ntuple descriptor.
virtual void CommitSealedPageV(std::span< RPageStorage::RSealedPageGroup > ranges)=0
Write a vector of preprocessed pages to storage. The corresponding columns must have been added befor...
std::unique_ptr< RNTupleWriteOptions > fOptions
RSealedPage SealPage(const RPage &page, const RColumnElementBase &element)
Helper for streaming a page.
Keeps track of the requested physical column IDs and their in-memory target type via a column element...
bool HasColumnInfos(DescriptorId_t physicalColumnId) const
const std::vector< RColumnInfo > & GetColumnInfos(DescriptorId_t physicalColumnId) const
std::unordered_map< DescriptorId_t, std::vector< RColumnInfo > > fColumnInfos
Maps physical column IDs to all the requested in-memory representations.
An RAII wrapper used for the writable access to RPageSource::fDescriptor. See GetSharedDescriptorGuar...
RExclDescriptorGuard(RNTupleDescriptor &desc, std::shared_mutex &lock)
RExclDescriptorGuard(const RExclDescriptorGuard &)=delete
RExclDescriptorGuard & operator=(RExclDescriptorGuard &&)=delete
RExclDescriptorGuard & operator=(const RExclDescriptorGuard &)=delete
An RAII wrapper used for the read-only access to RPageSource::fDescriptor. See GetExclDescriptorGuard...
RSharedDescriptorGuard & operator=(RSharedDescriptorGuard &&)=delete
RSharedDescriptorGuard(const RSharedDescriptorGuard &)=delete
RSharedDescriptorGuard(const RNTupleDescriptor &desc, std::shared_mutex &lock)
RSharedDescriptorGuard & operator=(const RSharedDescriptorGuard &)=delete
Abstract interface to read data from an ntuple.
virtual void LoadSealedPage(DescriptorId_t physicalColumnId, RClusterIndex clusterIndex, RSealedPage &sealedPage)=0
Read the packed and compressed bytes of a page into the memory buffer provided by sealedPage.
RPagePool fPagePool
Pages that are unzipped with IMT are staged into the page pool.
EPageStorageType GetType() final
Whether the concrete implementation is a sink or a source.
RPageSource(const RPageSource &)=delete
RPageSource & operator=(RPageSource &&)=delete
std::unique_ptr< RCounters > fCounters
RExclDescriptorGuard GetExclDescriptorGuard()
Note that the underlying lock is not recursive. See GetSharedDescriptorGuard() for further informatio...
RActivePhysicalColumns fActivePhysicalColumns
The active columns are implicitly defined by the model fields or views.
RPageSource & operator=(const RPageSource &)=delete
virtual RNTupleDescriptor AttachImpl()=0
LoadStructureImpl() has been called before AttachImpl() is called
const RNTupleReadOptions & GetReadOptions() const
virtual std::vector< std::unique_ptr< RCluster > > LoadClusters(std::span< RCluster::RKey > clusterKeys)=0
Populates all the pages of the given cluster ids and columns; it is possible that some columns do not...
REntryRange fEntryRange
Used by the cluster pool to prevent reading beyond the given range.
virtual RPageRef LoadPageImpl(ColumnHandle_t columnHandle, const RClusterInfo &clusterInfo, ClusterSize_t::ValueType idxInCluster)=0
virtual std::unique_ptr< RPageSource > CloneImpl() const =0
Returns a new, unattached page source for the same data set.
const RSharedDescriptorGuard GetSharedDescriptorGuard() const
Takes the read lock for the descriptor.
The interface of a task scheduler to schedule page (de)compression tasks.
virtual void Wait()=0
Blocks until all scheduled tasks finished.
virtual void AddTask(const std::function< void(void)> &taskFunc)=0
Take a callable that represents a task.
Common functionality of an ntuple storage for both reading and writing.
std::deque< RSealedPage > SealedPageSequence_t
std::unique_ptr< RPageAllocator > fPageAllocator
For the time being, we will use the heap allocator for all sources and sinks. This may change in the ...
virtual ColumnHandle_t AddColumn(DescriptorId_t fieldId, RColumn &column)=0
Register a new column.
static constexpr std::size_t kNBytesPageChecksum
The page checksum is a 64bit xxhash3.
virtual void DropColumn(ColumnHandle_t columnHandle)=0
Unregisters a column.
virtual Detail::RNTupleMetrics & GetMetrics()
Returns the default metrics object.
const std::string & GetNTupleName() const
Returns the NTuple name.
virtual EPageStorageType GetType()=0
Whether the concrete implementation is a sink or a source.
RPageStorage & operator=(RPageStorage &&other)=default
RPageStorage & operator=(const RPageStorage &other)=delete
RPageStorage(const RPageStorage &other)=delete
RPageStorage(RPageStorage &&other)=default
void SetTaskScheduler(RTaskScheduler *taskScheduler)
RColumnHandle ColumnHandle_t
The column handle identifies a column with the current open page storage.
ColumnId_t GetColumnId(ColumnHandle_t columnHandle) const
A page is a slice of a column that is mapped into memory.
Definition RPage.hxx:46
Helper to maintain a memory budget for the write pages of a set of columns.
bool TryUpdate(RColumn &column, std::size_t newWritePageSize)
Try to register the new write page size for the given column.
std::size_t fCurrentAllocatedBytes
Sum of all the write page sizes (their capacity) of the columns in fColumnsSortedByPageSize
bool TryEvict(std::size_t targetAvailableSize, std::size_t pageSizeLimit)
Flush columns in order of allocated write page size until the sum of all write page allocations leave...
std::size_t fMaxAllocatedBytes
Maximum allowed value for fCurrentAllocatedBytes, set from RNTupleWriteOptions::fPageBufferBudget.
std::set< RColumnInfo, std::greater< RColumnInfo > > fColumnsSortedByPageSize
All columns that called ReservePage() (hence TryUpdate()) at least once, sorted by their current writ...
Records the partition of data into pages for a particular column in a particular cluster.
Meta-data for a set of ntuple clusters.
Addresses a column element or field item relative to a particular cluster, instead of a global NTuple...
Base class for all ROOT issued exceptions.
Definition RError.hxx:78
Field specific extra type information from the header / extenstion header.
The on-storage meta-data of an ntuple.
The RNTupleModel encapulates the schema of an ntuple.
Common user-tunable settings for reading ntuples.
Common user-tunable settings for storing ntuples.
The class is used as a return type for operations that can fail; wraps a value of type T or an RError...
Definition RError.hxx:194
std::uint64_t NTupleSize_t
Integer type long enough to hold the maximum number of entries in a column.
std::uint64_t DescriptorId_t
Distriniguishes elements of the same type within a descriptor, e.g. different fields.
constexpr NTupleSize_t kInvalidNTupleIndex
std::int64_t ColumnId_t
Uniquely identifies a physical column within the scope of the current process, used to tag pages.
constexpr ClusterSize_t kInvalidClusterIndex(std::uint64_t(-1))
constexpr DescriptorId_t kInvalidDescriptorId
tbb::task_arena is an alias of tbb::interface7::task_arena, which doesn't allow to forward declare tb...
The identifiers that specifies the content of a (partial) cluster.
Definition RCluster.hxx:156
Every concrete RColumnElement type is identified by its on-disk type (column type) and the in-memory ...
The incremental changes to a RNTupleModel
Default I/O performance counters that get registered in fMetrics.
Detail::RNTupleTickCounter< Detail::RNTupleAtomicCounter > & fTimeCpuZip
Detail::RNTupleTickCounter< Detail::RNTupleAtomicCounter > & fTimeCpuWrite
Set of optional features supported by the persistent sink.
const RColumnElementBase * fElement
Corresponds to the page's elements, for size calculation etc.
void * fBuffer
Location for sealed output. The memory buffer has to be large enough.
bool fAllowAlias
If false, the output buffer must not point to the input page buffer, which would otherwise be an opti...
int fCompressionSetting
Compression algorithm and level to apply.
bool fWriteChecksum
Adds a 8 byte little-endian xxhash3 checksum to the page payload.
Cluster that was staged, but not yet logically appended to the RNTuple.
Summarizes cluster-level information that are necessary to load a certain page.
RClusterDescriptor::RPageRange::RPageInfoExtended fPageInfo
Location of the page on disk.
Default I/O performance counters that get registered in fMetrics
Detail::RNTupleTickCounter< Detail::RNTupleAtomicCounter > & fTimeCpuUnzip
Detail::RNTupleTickCounter< Detail::RNTupleAtomicCounter > & fTimeCpuRead
A range of sealed pages referring to the same column that can be used for vector commit.
RSealedPageGroup(DescriptorId_t d, SealedPageSequence_t::const_iterator b, SealedPageSequence_t::const_iterator e)
A sealed page contains the bytes of a page as written to storage (packed & compressed).
RResult< std::uint64_t > GetChecksum() const
Returns a failure if the sealed page has no checksum.
bool fHasChecksum
If set, the last 8 bytes of the buffer are the xxhash of the rest of the buffer.
std::size_t fBufferSize
Size of the page payload and the trailing checksum (if available)
RSealedPage(const void *buffer, std::size_t bufferSize, std::uint32_t nElements, bool hasChecksum=false)
RSealedPage & operator=(const RSealedPage &other)=default
RSealedPage & operator=(RSealedPage &&other)=default
We do not need to store the element size / uncompressed page size because we know to which column the...
Wrap the integer in a struct in order to avoid template specialization clash with std::uint64_t.
Generic information about the physical location of data.