1
0
mirror of https://github.com/mariadb-corporation/mariadb-columnstore-engine.git synced 2025-12-24 14:20:59 +03:00

No boost condition (#2822)

This patch replaces boost primitives with stdlib counterparts.
This commit is contained in:
Leonid Fedorov
2023-04-22 00:42:45 +03:00
committed by GitHub
parent 3ce19abdae
commit f916e64927
245 changed files with 1261 additions and 2007 deletions

View File

@@ -71,7 +71,7 @@ const std::string ERR_LOG_SUFFIX = ".err"; // Job err log file suffix
namespace WriteEngine
{
/* static */ boost::ptr_vector<TableInfo> BulkLoad::fTableInfo;
/* static */ boost::mutex* BulkLoad::fDDLMutex = 0;
/* static */ std::mutex* BulkLoad::fDDLMutex = 0;
/* static */ const std::string BulkLoad::DIR_BULK_JOB("job");
/* static */ const std::string BulkLoad::DIR_BULK_TEMP_JOB("tmpjob");
@@ -165,7 +165,7 @@ BulkLoad::BulkLoad()
fTableInfo.clear();
setDebugLevel(DEBUG_0);
fDDLMutex = new boost::mutex();
fDDLMutex = new std::mutex();
memset(&fStartTime, 0, sizeof(timeval));
memset(&fEndTime, 0, sizeof(timeval));
}
@@ -1584,7 +1584,7 @@ int BulkLoad::updateNextValue(OID columnOid, uint64_t nextAutoIncVal)
// job for 2 tables; so we put a mutex here just in case the DDLClient code
// won't work well with 2 competing WE_DDLCommandClient objects in the same
// process (ex: if there is any static data in WE_DDLCommandClient).
boost::mutex::scoped_lock lock(*fDDLMutex);
std::unique_lock lock(*fDDLMutex);
WE_DDLCommandClient ddlCommandClt;
unsigned int rc = ddlCommandClt.UpdateSyscolumnNextval(columnOid, nextAutoIncVal);

View File

@@ -40,7 +40,8 @@
#include "brmtypes.h"
#include "boost/ptr_container/ptr_vector.hpp"
#include <boost/thread/thread.hpp>
#include <boost/thread/mutex.hpp>
#include <map>
#include <mutex>
#include <boost/bind.hpp>
#include <boost/scoped_ptr.hpp>
#include <boost/uuid/uuid.hpp>
@@ -199,9 +200,9 @@ class BulkLoad : public FileOp
int fNoOfReadThreads; // Number of read threads
boost::thread_group fReadThreads; // Read thread group
boost::thread_group fParseThreads; // Parse thread group
boost::mutex fReadMutex; // Manages table selection by each
std::mutex fReadMutex; // Manages table selection by each
// read thread
boost::mutex fParseMutex; // Manages table/buffer/column
std::mutex fParseMutex; // Manages table/buffer/column
// selection by each parsing thread
BRM::TxnID fTxnID; // TransID acquired from SessionMgr
bool fKeepRbMetaFiles; // Keep/delete bulkRB metadata files
@@ -218,7 +219,7 @@ class BulkLoad : public FileOp
ImportDataMode fImportDataMode; // Importing text or binary data
bool fbContinue; // true when read and parse r running
//
static boost::mutex* fDDLMutex; // Insure only 1 DDL op at a time
static std::mutex* fDDLMutex; // Insure only 1 DDL op at a time
EXPORT static const std::string DIR_BULK_JOB; // Bulk job directory
EXPORT static const std::string DIR_BULK_TEMP_JOB; // Dir for tmp job files

View File

@@ -1536,7 +1536,7 @@ int BulkLoadBuffer::parse(ColumnInfo& columnInfo)
// variables). It should be okay to reference a copy of these variables
// as no other thread should be changing them while we are in parse().
{
boost::mutex::scoped_lock lock(fSyncUpdatesBLB);
std::unique_lock lock(fSyncUpdatesBLB);
fTotalReadRowsParser = fTotalReadRows;
fStartRowParser = fStartRow;
fDataParser = fData;
@@ -2035,7 +2035,7 @@ int BulkLoadBuffer::fillFromMemory(const BulkLoadBuffer& overFlowBufIn, const ch
const boost::ptr_vector<ColumnInfo>& columnsInfo,
unsigned int allowedErrCntThisCall)
{
boost::mutex::scoped_lock lock(fSyncUpdatesBLB);
std::unique_lock lock(fSyncUpdatesBLB);
reset();
copyOverflow(overFlowBufIn);
size_t readSize = 0;
@@ -2138,7 +2138,7 @@ int BulkLoadBuffer::fillFromFile(const BulkLoadBuffer& overFlowBufIn, FILE* hand
RID& correctTotalRows, const boost::ptr_vector<ColumnInfo>& columnsInfo,
unsigned int allowedErrCntThisCall)
{
boost::mutex::scoped_lock lock(fSyncUpdatesBLB);
std::unique_lock lock(fSyncUpdatesBLB);
reset();
copyOverflow(overFlowBufIn);
size_t readSize = 0;

View File

@@ -137,7 +137,7 @@ class BulkLoadBuffer
char fColDelim; // Character to delimit columns in a row
unsigned fBufferSize; // Size of input read buffer (fData)
unsigned fReadSize; // Number of bytes in read buffer(fData)
boost::mutex fSyncUpdatesBLB; // Mutex to synchronize updates
std::mutex fSyncUpdatesBLB; // Mutex to synchronize updates
Log* fLog; // Logger object
bool fNullStringMode; // Indicates if "NULL" string is to be
// treated as a NULL value or not

View File

@@ -31,6 +31,8 @@
* ColumnBufferCompressed.
*/
#include "we_colbufmgr.h"
#include "we_colbuf.h"
#include "we_colbufcompressed.h"
@@ -38,8 +40,10 @@
#include "we_bulkstatus.h"
#include "we_log.h"
#include "blocksize.h"
#include <chrono>
#include <sstream>
#include <boost/date_time/posix_time/posix_time_types.hpp>
namespace
{
@@ -123,9 +127,9 @@ int ColumnBufferManager::reserveSection(RID startRowId, uint32_t nRowsIn, uint32
Stats::startParseEvent(WE_STATS_WAIT_TO_RESERVE_OUT_BUF);
#endif
*cbs = 0;
boost::posix_time::seconds wait_seconds(COND_WAIT_SECONDS);
std::chrono::seconds wait_seconds(COND_WAIT_SECONDS);
boost::mutex::scoped_lock lock(fColInfo->colMutex());
std::unique_lock lock(fColInfo->colMutex());
//..Ensure that ColumnBufferSection allocations are made in input row order
bool bWaitedForInSequence = false;
@@ -145,7 +149,7 @@ int ColumnBufferManager::reserveSection(RID startRowId, uint32_t nRowsIn, uint32
fLog->logMsg(oss.str(), MSGLVL_INFO2);
}
fOutOfSequence.timed_wait(lock, wait_seconds);
fOutOfSequence.wait_for(lock, wait_seconds);
// See if JobStatus has been set to terminate by another thread
if (BulkStatus::getJobStatus() == EXIT_FAILURE)
@@ -179,7 +183,7 @@ int ColumnBufferManager::reserveSection(RID startRowId, uint32_t nRowsIn, uint32
fLog->logMsg(oss.str(), MSGLVL_INFO2);
}
fResizeInProgress.timed_wait(lock, wait_seconds);
fResizeInProgress.wait_for(lock, wait_seconds);
// See if JobStatus has been set to terminate by another thread
if (BulkStatus::getJobStatus() == EXIT_FAILURE)
@@ -253,7 +257,7 @@ int ColumnBufferManager::reserveSection(RID startRowId, uint32_t nRowsIn, uint32
fLog->logMsg(oss.str(), MSGLVL_INFO2);
}
fBufInUse.timed_wait(lock, wait_seconds);
fBufInUse.wait_for(lock, wait_seconds);
// See if JobStatus has been set to quit by another thread
if (BulkStatus::getJobStatus() == EXIT_FAILURE)
@@ -330,7 +334,7 @@ int ColumnBufferManager::releaseSection(ColumnBufferSection* cbs)
#ifdef PROFILE
Stats::startParseEvent(WE_STATS_WAIT_TO_RELEASE_OUT_BUF);
#endif
boost::mutex::scoped_lock lock(fColInfo->colMutex());
std::unique_lock lock(fColInfo->colMutex());
#ifdef PROFILE
Stats::stopParseEvent(WE_STATS_WAIT_TO_RELEASE_OUT_BUF);
#endif
@@ -675,8 +679,8 @@ int ColumnBufferManager::flush()
//------------------------------------------------------------------------------
int ColumnBufferManager::intermediateFlush()
{
boost::posix_time::seconds wait_seconds(COND_WAIT_SECONDS);
boost::mutex::scoped_lock lock(fColInfo->colMutex());
std::chrono::seconds wait_seconds(COND_WAIT_SECONDS);
std::unique_lock lock(fColInfo->colMutex());
// Wait for all other threads which are currently parsing rows,
// to finish parsing the data in those sections.
@@ -686,7 +690,7 @@ int ColumnBufferManager::intermediateFlush()
while (fSectionsInUse.size() > 0)
{
fBufInUse.timed_wait(lock, wait_seconds);
fBufInUse.wait_for(lock, wait_seconds);
// See if JobStatus has been set to terminate by another thread
if (BulkStatus::getJobStatus() == EXIT_FAILURE)
@@ -729,7 +733,7 @@ int ColumnBufferManager::rowsExtentCheck(int nRows, int& nRows2)
//------------------------------------------------------------------------------
int ColumnBufferManager::extendTokenColumn()
{
boost::mutex::scoped_lock lock(fColInfo->colMutex());
std::unique_lock lock(fColInfo->colMutex());
return fColInfo->extendColumn(false);
}

View File

@@ -26,8 +26,9 @@
#pragma once
#include <boost/thread/mutex.hpp>
#include <boost/thread/condition.hpp>
#include <map>
#include <mutex>
#include <condition_variable>
#include <list>
#include <vector>
@@ -223,17 +224,17 @@ class ColumnBufferManager
/** @brief Condition variable for threads waiting for resize to complete
*/
boost::condition fResizeInProgress;
std::condition_variable fResizeInProgress;
/** @brief Condition variable for threads waiting for all buffer sections
* to be released
*/
boost::condition fBufInUse;
std::condition_variable fBufInUse;
/** @brief Condition variable for threads who have arrived out-of-
* sequence with respect to their row-id
*/
boost::condition fOutOfSequence;
std::condition_variable fOutOfSequence;
/** @brief Width of the column
*/

View File

@@ -53,7 +53,7 @@ namespace WriteEngine
// @bug 4806: Added bIsNewExtent; Set CP min/max for very first extent on a PM
void ColExtInf::addFirstEntry(RID lastInputRow, BRM::LBID_t lbid, bool bIsNewExtent)
{
boost::mutex::scoped_lock lock(fMapMutex);
std::unique_lock lock(fMapMutex);
ColExtInfEntry entry(lbid, bIsNewExtent);
fMap[lastInputRow] = entry;
@@ -70,7 +70,7 @@ template <typename T>
void ColExtInf::addOrUpdateEntryTemplate(RID lastInputRow, T minVal, T maxVal, ColDataType colDataType,
int width)
{
boost::mutex::scoped_lock lock(fMapMutex);
std::unique_lock lock(fMapMutex);
RowExtMap::iterator iter = fMap.find(lastInputRow);
@@ -152,7 +152,7 @@ void ColExtInf::addOrUpdateEntryTemplate(RID lastInputRow, T minVal, T maxVal, C
//------------------------------------------------------------------------------
int ColExtInf::updateEntryLbid(BRM::LBID_t startLbid)
{
boost::mutex::scoped_lock lock(fMapMutex);
std::unique_lock lock(fMapMutex);
// fPendingExtentRows is a Set carrying a sorted list of the last Row
// number in each extent. We should be allocating/assigning LBIDs in
@@ -190,7 +190,7 @@ void ColExtInf::getCPInfoForBRM(JobColumn column, BRMReporter& brmReporter)
{
bool bIsChar = ((column.weType == WriteEngine::WR_CHAR) && (column.colType != COL_TYPE_DICT));
boost::mutex::scoped_lock lock(fMapMutex);
std::unique_lock lock(fMapMutex);
RowExtMap::const_iterator iter = fMap.begin();
@@ -280,7 +280,7 @@ void ColExtInf::getCPInfoForBRM(JobColumn column, BRMReporter& brmReporter)
//------------------------------------------------------------------------------
void ColExtInf::print(const JobColumn& column)
{
boost::mutex::scoped_lock lock(fMapMutex);
std::unique_lock lock(fMapMutex);
bool bIsChar = ((column.weType == WriteEngine::WR_CHAR) && (column.colType != COL_TYPE_DICT));
std::ostringstream oss;
oss << "ColExtInf Map for OID: " << fColOid;

View File

@@ -31,7 +31,8 @@
#include <stdint.h>
#include <set>
#include <tr1/unordered_map>
#include <boost/thread/mutex.hpp>
#include <map>
#include <mutex>
#include "brmtypes.h"
#include "we_type.h"
@@ -242,7 +243,7 @@ class ColExtInf : public ColExtInfBase
private:
OID fColOid; // Column OID for the relevant extents
Log* fLog; // Log used for debug logging
boost::mutex fMapMutex; // protects unordered map access
std::mutex fMapMutex; // protects unordered map access
std::set<RID> fPendingExtentRows; // list of lastInputRow entries that
// are awaiting an LBID assignment.

View File

@@ -118,7 +118,7 @@ int ColumnAutoInc::init(const std::string& fullTableName, ColumnInfo* colInfo)
//------------------------------------------------------------------------------
void ColumnAutoInc::initNextAutoInc(uint64_t nextValue)
{
boost::mutex::scoped_lock lock(fAutoIncMutex);
std::unique_lock lock(fAutoIncMutex);
// nextValue is unusable if < 1; probably means we already reached max value
if (nextValue < 1)
@@ -227,7 +227,7 @@ uint64_t ColumnAutoInc::getNextAutoIncToSave()
{
uint64_t nextValue = AUTOINCR_SATURATED;
boost::mutex::scoped_lock lock(fAutoIncMutex);
std::unique_lock lock(fAutoIncMutex);
// nextValue is returned as -1 if we reached max value
if (fAutoIncLastValue < fMaxIntSat)
@@ -314,7 +314,7 @@ ColumnAutoIncJob::~ColumnAutoIncJob()
/* virtual */
int ColumnAutoIncJob::reserveNextRange(uint32_t autoIncCount, uint64_t& nextValue)
{
boost::mutex::scoped_lock lock(fAutoIncMutex);
std::unique_lock lock(fAutoIncMutex);
if ((fMaxIntSat - autoIncCount) < fAutoIncLastValue)
{
@@ -376,7 +376,7 @@ int ColumnAutoIncIncremental::reserveNextRange(uint32_t autoIncCount, uint64_t&
// processing AI ranges out of order, so we don't arbitrarily
// update fAutoIncLastValue. We only update it if the range in question
// exceeds the current value for fAutoIncLastValue.
boost::mutex::scoped_lock lock(fAutoIncMutex);
std::unique_lock lock(fAutoIncMutex);
if (autoIncLastValue > fAutoIncLastValue)
fAutoIncLastValue = autoIncLastValue;

View File

@@ -26,7 +26,8 @@
#pragma once
#include <string>
#include <boost/thread/mutex.hpp>
#include <map>
#include <mutex>
#include <boost/scoped_ptr.hpp>
#include "we_type.h"
@@ -74,7 +75,7 @@ class ColumnAutoInc
int getNextValueFromSysCat(uint64_t& nextValue);
Log* fLog; // import log file
boost::mutex fAutoIncMutex; // Mutex to manage fAutoIncLastValue
std::mutex fAutoIncMutex; // Mutex to manage fAutoIncLastValue
uint64_t fAutoIncLastValue; // Tracks latest autoincrement value used
uint64_t fMaxIntSat; // Maximum saturation value
std::string fTableName; // Full table name (schema.table) for AI column

View File

@@ -301,7 +301,7 @@ int ColumnInfo::createDelayedFileIfNeeded(const std::string& tableName)
// No sense in waiting for a fColMutex lock, when 99.99% of the time,
// all we need to do is check fDelayedFileCreation, see that it's value
// is INITIAL_DBFILE_STAT_FILE_EXISTS, and exit the function.
boost::mutex::scoped_lock lock(fDelayedFileCreateMutex);
std::unique_lock lock(fDelayedFileCreateMutex);
if (fDelayedFileCreation == INITIAL_DBFILE_STAT_FILE_EXISTS)
return NO_ERROR;
@@ -323,7 +323,7 @@ int ColumnInfo::createDelayedFileIfNeeded(const std::string& tableName)
// fDelayedFileCreateMutex lock might suffice, but better to explicitly
// lock fColMutex since we are modifying attributes that we typically
// change within the scope of a fColMutex lock.
boost::mutex::scoped_lock lock2(fColMutex);
std::unique_lock lock2(fColMutex);
uint16_t dbRoot = curCol.dataFile.fDbRoot;
uint32_t partition = curCol.dataFile.fPartition;
@@ -1116,7 +1116,7 @@ int ColumnInfo::finishParsing()
// thread working on this column. But, we use the mutex to insure that
// we see the latest state that may have been set by another parsing thread
// working with the same column.
boost::mutex::scoped_lock lock(fColMutex);
std::unique_lock lock(fColMutex);
// Force the flushing of remaining data in the output buffer
if (fColBufferMgr)
@@ -1165,7 +1165,7 @@ int ColumnInfo::finishParsing()
//------------------------------------------------------------------------------
void ColumnInfo::getBRMUpdateInfo(BRMReporter& brmReporter)
{
boost::mutex::scoped_lock lock(fColMutex);
std::unique_lock lock(fColMutex);
// Useful for debugging
// printCPInfo(column);
@@ -1495,7 +1495,7 @@ int ColumnInfo::finishAutoInc()
//------------------------------------------------------------------------------
void ColumnInfo::getSegFileInfo(DBRootExtentInfo& fileInfo)
{
boost::mutex::scoped_lock lock(fColMutex);
std::unique_lock lock(fColMutex);
fileInfo.fDbRoot = curCol.dataFile.fDbRoot;
fileInfo.fPartition = curCol.dataFile.fPartition;
fileInfo.fSegment = curCol.dataFile.fSegment;
@@ -1692,7 +1692,7 @@ int ColumnInfo::updateDctnryStore(char* buf, ColPosPair** pos, const int totalRo
#ifdef PROFILE
Stats::startParseEvent(WE_STATS_WAIT_TO_PARSE_DCT);
#endif
boost::mutex::scoped_lock lock(fDictionaryMutex);
std::unique_lock lock(fDictionaryMutex);
#ifdef PROFILE
Stats::stopParseEvent(WE_STATS_WAIT_TO_PARSE_DCT);
#endif

View File

@@ -33,7 +33,8 @@
#include "we_colextinf.h"
#include "we_dctnrycompress.h"
#include <boost/thread/mutex.hpp>
#include <map>
#include <mutex>
#include <boost/scoped_ptr.hpp>
#include <sys/time.h>
#include <vector>
@@ -377,7 +378,7 @@ class ColumnInfo : public WeUIDGID
* This was formerly the fMgrMutex in ColumnBufferManager. See comments
* that precede this class definition for more information.
*/
boost::mutex& colMutex();
std::mutex& colMutex();
/** @brief Get number of rows per extent
*/
@@ -424,10 +425,10 @@ class ColumnInfo : public WeUIDGID
// Protected Data Members
//--------------------------------------------------------------------------
boost::mutex fDictionaryMutex; // Mutex for dicionary updates
boost::mutex fColMutex; // Mutex for column changes
boost::mutex fAutoIncMutex; // Mutex to manage fAutoIncLastValue
boost::mutex fDelayedFileCreateMutex; // Manage delayed file check/create
std::mutex fDictionaryMutex; // Mutex for dicionary updates
std::mutex fColMutex; // Mutex for column changes
std::mutex fAutoIncMutex; // Mutex to manage fAutoIncLastValue
std::mutex fDelayedFileCreateMutex; // Manage delayed file check/create
Log* fLog; // Object used for logging
// Blocks to skip at start of bulk load, if starting file has to be created
@@ -500,7 +501,7 @@ inline void ColumnInfo::setUIDGID(const uid_t p_uid, const gid_t p_gid)
colOp->setUIDGID(this);
}
inline boost::mutex& ColumnInfo::colMutex()
inline std::mutex& ColumnInfo::colMutex()
{
return fColMutex;
}

View File

@@ -67,7 +67,7 @@ ExtentStripeAlloc::~ExtentStripeAlloc()
//------------------------------------------------------------------------------
void ExtentStripeAlloc::addColumn(OID colOID, int colWidth)
{
boost::mutex::scoped_lock lock(fMapMutex);
std::unique_lock lock(fMapMutex);
fColOIDs.push_back(colOID);
fColWidths.push_back(colWidth);
@@ -108,7 +108,7 @@ int ExtentStripeAlloc::allocateExtent(OID oid, uint16_t dbRoot,
std::pair<AllocExtMapIter, AllocExtMapIter> iters;
boost::mutex::scoped_lock lock(fMapMutex);
std::unique_lock lock(fMapMutex);
// Search for an extent matching the requested OID and DBRoot.
// We also filter by selecting the lowest stripe number. See
@@ -257,7 +257,7 @@ int ExtentStripeAlloc::allocateExtent(OID oid, uint16_t dbRoot,
//------------------------------------------------------------------------------
void ExtentStripeAlloc::print()
{
boost::mutex::scoped_lock lock(fMapMutex);
std::unique_lock lock(fMapMutex);
std::ostringstream oss;
oss << "Current Pending Extents for table " << fTableOID << ":";

View File

@@ -30,7 +30,8 @@
#include <vector>
#include <tr1/unordered_map>
#include <boost/thread/mutex.hpp>
#include <map>
#include <mutex>
#include "we_type.h"
#include "brmtypes.h"
@@ -156,7 +157,7 @@ class ExtentStripeAlloc
OID fTableOID; // Table extents to be allocated
Log* fLog; // Log used for debug logging
unsigned int fStripeCount; // Extent "stripe" counter
boost::mutex fMapMutex; // protects unordered map access
std::mutex fMapMutex; // protects unordered map access
std::vector<OID> fColOIDs; // Vector of column OIDs
std::vector<int> fColWidths; // Widths associated with fColOIDs

View File

@@ -245,7 +245,7 @@ void TableInfo::closeOpenDbFiles()
//------------------------------------------------------------------------------
bool TableInfo::lockForRead(const int& locker)
{
boost::mutex::scoped_lock lock(fSyncUpdatesTI);
std::unique_lock lock(fSyncUpdatesTI);
if (fLocker == -1)
{
@@ -278,7 +278,7 @@ int TableInfo::readTableData()
if (rc != NO_ERROR)
{
// Mark the table status as error and exit.
boost::mutex::scoped_lock lock(fSyncUpdatesTI);
std::unique_lock lock(fSyncUpdatesTI);
fStatusTI = WriteEngine::ERR;
return rc;
}
@@ -314,7 +314,7 @@ int TableInfo::readTableData()
// See if JobStatus has been set to terminate by another thread
if (BulkStatus::getJobStatus() == EXIT_FAILURE)
{
boost::mutex::scoped_lock lock(fSyncUpdatesTI);
std::unique_lock lock(fSyncUpdatesTI);
fStartTime = readStart;
fStatusTI = WriteEngine::ERR;
its.msg_type = ImportTeleStats::IT_TERM;
@@ -350,7 +350,7 @@ int TableInfo::readTableData()
// See if JobStatus has been set to terminate by another thread
if (BulkStatus::getJobStatus() == EXIT_FAILURE)
{
boost::mutex::scoped_lock lock(fSyncUpdatesTI);
std::unique_lock lock(fSyncUpdatesTI);
fStartTime = readStart;
fStatusTI = WriteEngine::ERR;
its.msg_type = ImportTeleStats::IT_TERM;
@@ -437,7 +437,7 @@ int TableInfo::readTableData()
// need to exit.
// mark the table status as error and exit.
{
boost::mutex::scoped_lock lock(fSyncUpdatesTI);
std::unique_lock lock(fSyncUpdatesTI);
fStartTime = readStart;
fStatusTI = WriteEngine::ERR;
fBuffers[readBufNo].setStatusBLB(WriteEngine::ERR);
@@ -491,7 +491,7 @@ int TableInfo::readTableData()
// number of errors > maximum allowed. hence return error.
{
boost::mutex::scoped_lock lock(fSyncUpdatesTI);
std::unique_lock lock(fSyncUpdatesTI);
fStartTime = readStart;
fStatusTI = WriteEngine::ERR;
fBuffers[readBufNo].setStatusBLB(WriteEngine::ERR);
@@ -518,7 +518,7 @@ int TableInfo::readTableData()
#ifdef PROFILE
Stats::startReadEvent(WE_STATS_WAIT_TO_COMPLETE_READ);
#endif
boost::mutex::scoped_lock lock(fSyncUpdatesTI);
std::unique_lock lock(fSyncUpdatesTI);
#ifdef PROFILE
Stats::stopReadEvent(WE_STATS_WAIT_TO_COMPLETE_READ);
Stats::startReadEvent(WE_STATS_COMPLETING_READ);
@@ -629,7 +629,7 @@ void TableInfo::writeErrorList(const std::vector<std::pair<RID, std::string> >*
if ((errorRowsCount > 0) || (errorDatRowsCount > 0) || (bCloseFile))
{
boost::mutex::scoped_lock lock(fErrorRptInfoMutex);
std::unique_lock lock(fErrorRptInfoMutex);
if ((errorRowsCount > 0) || (bCloseFile))
writeErrReason(errorRows, bCloseFile);
@@ -674,7 +674,7 @@ int TableInfo::parseColumn(const int& columnId, const int& bufferId, double& pro
//------------------------------------------------------------------------------
int TableInfo::setParseComplete(const int& columnId, const int& bufferId, double processingTime)
{
boost::mutex::scoped_lock lock(fSyncUpdatesTI);
std::unique_lock lock(fSyncUpdatesTI);
// Check table status in case race condition results in this function
// being called after fStatusTI was set to ERR by another thread.
@@ -1051,7 +1051,7 @@ int TableInfo::finishBRM()
std::vector<std::string>* errFiles = 0;
std::vector<std::string>* badFiles = 0;
{
boost::mutex::scoped_lock lock(fErrorRptInfoMutex);
std::unique_lock lock(fErrorRptInfoMutex);
errFiles = &fErrFiles;
badFiles = &fBadFiles;
}
@@ -1071,7 +1071,7 @@ int TableInfo::finishBRM()
//------------------------------------------------------------------------------
void TableInfo::setParseError()
{
boost::mutex::scoped_lock lock(fSyncUpdatesTI);
std::unique_lock lock(fSyncUpdatesTI);
fStatusTI = WriteEngine::ERR;
}
@@ -1084,7 +1084,7 @@ void TableInfo::setParseError()
// Added report parm and couts below.
int TableInfo::getColumnForParse(const int& id, const int& bufferId, bool report)
{
boost::mutex::scoped_lock lock(fSyncUpdatesTI);
std::unique_lock lock(fSyncUpdatesTI);
double maxTime = 0;
int columnId = -1;
@@ -1360,7 +1360,7 @@ void TableInfo::closeTableFile()
// Added report parm and couts below.
bool TableInfo::isBufferAvailable(bool report)
{
boost::mutex::scoped_lock lock(fSyncUpdatesTI);
std::unique_lock lock(fSyncUpdatesTI);
Status bufferStatus = fBuffers[fCurrentReadBuffer].getStatusBLB();
if ((bufferStatus == WriteEngine::PARSE_COMPLETE) || (bufferStatus == WriteEngine::NEW))

View File

@@ -26,7 +26,8 @@
#include <utility>
#include <vector>
#include <boost/thread/mutex.hpp>
#include <map>
#include <mutex>
#include <boost/ptr_container/ptr_vector.hpp>
#include <boost/uuid/uuid.hpp>
@@ -105,9 +106,9 @@ class TableInfo : public WeUIDGID
* getting/setting the status of the BulkLoadBuffer objects, so
* fSyncUpdatesTI is also used to set/get the BulkLoadBuffer status.
*/
boost::mutex fSyncUpdatesTI;
std::mutex fSyncUpdatesTI;
boost::mutex fErrorRptInfoMutex; // Used to synhronize access to
std::mutex fErrorRptInfoMutex; // Used to synhronize access to
// fRejectDataFile & fRejectErrFile
int fLocker; // Read thread id reading this table
std::vector<std::string> fLoadFileList; // Load files
@@ -540,7 +541,7 @@ inline bool TableInfo::isTableLocked()
inline void TableInfo::markTableComplete()
{
boost::mutex::scoped_lock lock(fSyncUpdatesTI);
std::unique_lock lock(fSyncUpdatesTI);
fStatusTI = WriteEngine::PARSE_COMPLETE;
}

View File

@@ -166,7 +166,7 @@ void BulkLoad::read(int id)
//------------------------------------------------------------------------------
int BulkLoad::lockTableForRead(int id)
{
boost::mutex::scoped_lock lock(fReadMutex);
std::unique_lock lock(fReadMutex);
for (unsigned i = 0; i < fTableInfo.size(); ++i)
{
@@ -315,7 +315,7 @@ void BulkLoad::parse(int id)
#ifdef PROFILE
Stats::startParseEvent(WE_STATS_WAIT_TO_COMPLETE_PARSE);
#endif
boost::mutex::scoped_lock lock(fParseMutex);
std::unique_lock lock(fParseMutex);
#ifdef PROFILE
Stats::stopParseEvent(WE_STATS_WAIT_TO_COMPLETE_PARSE);
Stats::startParseEvent(WE_STATS_COMPLETING_PARSE);
@@ -417,7 +417,7 @@ bool BulkLoad::lockColumnForParse(int thrdId, int& tableId, int& columnId, int&
// Check if the currentParseBuffer is available for parsing
// If yes, put the locker and fill the tableId and columnId
// else, go to the next table for checking if a column is available
boost::mutex::scoped_lock lock(fParseMutex);
std::unique_lock lock(fParseMutex);
for (unsigned i = 0; i < fTableInfo.size(); ++i)
{
@@ -498,7 +498,7 @@ void BulkLoad::setParseErrorOnTable(int tableId, bool lockParseMutex)
{
if (lockParseMutex)
{
boost::mutex::scoped_lock lock(fParseMutex);
std::unique_lock lock(fParseMutex);
fTableInfo[tableId].setParseError();
}
else

View File

@@ -32,8 +32,8 @@
#endif
using namespace std;
#include <boost/thread/mutex.hpp>
using namespace boost;
#include <map>
#include <mutex>
#include "messagequeue.h"
#include "bytestream.h"
@@ -225,7 +225,7 @@ void WEClients::Setup()
string fServer(buff);
boost::shared_ptr<MessageQueueClient> cl(new MessageQueueClient(fServer, rm->getConfig()));
boost::shared_ptr<boost::mutex> nl(new boost::mutex());
boost::shared_ptr<std::mutex> nl(new std::mutex());
// Bug 5224. Take out the retrys. If connection fails, we assume the server is down.
try
@@ -355,7 +355,7 @@ void WEClients::Listen(boost::shared_ptr<MessageQueueClient> client, uint32_t co
Error:
// error condition! push 0 length bs to messagequeuemap and
// eventually let jobstep error out.
boost::mutex::scoped_lock lk(fMlock);
std::unique_lock lk(fMlock);
MessageQueueMap::iterator map_tok;
sbs.reset(new ByteStream(0));
@@ -371,7 +371,7 @@ Error:
// reset the pmconnection map
{
boost::mutex::scoped_lock onErrLock(fOnErrMutex);
std::unique_lock onErrLock(fOnErrMutex);
string moduleName = client->moduleName();
ClientList::iterator itor = fPmConnections.begin();
@@ -396,13 +396,13 @@ void WEClients::addQueue(uint32_t key)
{
bool b;
boost::mutex* lock = new boost::mutex();
condition* cond = new condition();
std::mutex* lock = new std::mutex();
std::condition_variable* cond = new std::condition_variable();
boost::shared_ptr<MQE> mqe(new MQE(pmCount));
mqe->queue = WESMsgQueue(lock, cond);
boost::mutex::scoped_lock lk(fMlock);
std::unique_lock lk(fMlock);
b = fSessionMessages.insert(pair<uint32_t, boost::shared_ptr<MQE> >(key, mqe)).second;
if (!b)
@@ -415,7 +415,7 @@ void WEClients::addQueue(uint32_t key)
void WEClients::removeQueue(uint32_t key)
{
boost::mutex::scoped_lock lk(fMlock);
std::unique_lock lk(fMlock);
MessageQueueMap::iterator map_tok = fSessionMessages.find(key);
if (map_tok == fSessionMessages.end())
@@ -428,7 +428,7 @@ void WEClients::removeQueue(uint32_t key)
void WEClients::shutdownQueue(uint32_t key)
{
boost::mutex::scoped_lock lk(fMlock);
std::unique_lock lk(fMlock);
MessageQueueMap::iterator map_tok = fSessionMessages.find(key);
if (map_tok == fSessionMessages.end())
@@ -443,7 +443,7 @@ void WEClients::read(uint32_t key, SBS& bs)
boost::shared_ptr<MQE> mqe;
// Find the StepMsgQueueList for this session
boost::mutex::scoped_lock lk(fMlock);
std::unique_lock lk(fMlock);
MessageQueueMap::iterator map_tok = fSessionMessages.find(key);
if (map_tok == fSessionMessages.end())
@@ -521,7 +521,7 @@ void WEClients::addDataToOutput(SBS sbs, uint32_t connIndex)
*sbs >> uniqueId;
boost::shared_ptr<MQE> mqe;
boost::mutex::scoped_lock lk(fMlock);
std::unique_lock lk(fMlock);
MessageQueueMap::iterator map_tok = fSessionMessages.find(uniqueId);
if (map_tok == fSessionMessages.end())

View File

@@ -27,13 +27,13 @@
#include <string>
#include <map>
#include <boost/thread.hpp>
#include <boost/thread/condition.hpp>
#include <condition_variable>
#include <boost/scoped_array.hpp>
#include "bytestream.h"
//#include "we_message.h"
#include "threadsafequeue.h"
#include "rwlock_local.h"
#include "resourcemanager.h"
#define EXPORT
@@ -49,7 +49,7 @@ class WEClients
EXPORT WEClients(int PrgmID);
EXPORT ~WEClients();
// static boost::mutex map_mutex;
// static std::mutex map_mutex;
EXPORT void addQueue(uint32_t key);
EXPORT void removeQueue(uint32_t key);
EXPORT void shutdownQueue(uint32_t key);
@@ -153,14 +153,14 @@ class WEClients
ReaderList fWESReader; // all the reader threads for the pm servers
MessageQueueMap
fSessionMessages; // place to put messages from the pm server to be returned by the Read method
boost::mutex fMlock; // sessionMessages mutex
std::vector<boost::shared_ptr<boost::mutex> > fWlock; // WES socket write mutexes
std::mutex fMlock; // sessionMessages mutex
std::vector<boost::shared_ptr<std::mutex> > fWlock; // WES socket write mutexes
bool fBusy;
volatile uint32_t closingConnection;
uint32_t pmCount;
boost::mutex fOnErrMutex; // to lock function scope to reset pmconnections under error condition
std::mutex fOnErrMutex; // to lock function scope to reset pmconnections under error condition
boost::mutex ackLock;
std::mutex ackLock;
public:
enum

View File

@@ -66,7 +66,7 @@ using namespace execplan;
namespace redistribute
{
RedistributeControl* RedistributeControl::fInstance = NULL;
boost::mutex instanceMutex;
std::mutex instanceMutex;
const string RedistributeDir("/data1/systemFiles/redistribute");
const string InfoFileName("/redistribute.info");
@@ -75,7 +75,7 @@ const string PlanFileName("/redistribute.plan");
RedistributeControl* RedistributeControl::instance()
{
// The constructor is protected by instanceMutex lock.
boost::mutex::scoped_lock lock(instanceMutex);
std::unique_lock lock(instanceMutex);
if (fInstance == NULL)
fInstance = new RedistributeControl();
@@ -127,7 +127,7 @@ RedistributeControl::~RedistributeControl()
int RedistributeControl::handleUIMsg(messageqcpp::ByteStream& bs, messageqcpp::IOSocket& so)
{
boost::mutex::scoped_lock sessionLock(fSessionMutex);
std::unique_lock sessionLock(fSessionMutex);
uint32_t status = RED_STATE_UNDEF;
const RedistributeMsgHeader* h = (const RedistributeMsgHeader*)bs.buf();
@@ -379,7 +379,7 @@ uint32_t RedistributeControl::getCurrentState()
{
uint32_t status = RED_STATE_UNDEF;
ostringstream oss;
boost::mutex::scoped_lock lock(fInfoFileMutex);
std::unique_lock lock(fInfoFileMutex);
if (!fInfoFilePtr)
{
@@ -450,7 +450,7 @@ bool RedistributeControl::getStartOptions(messageqcpp::ByteStream& bs)
void RedistributeControl::updateState(uint32_t s)
{
boost::mutex::scoped_lock lock(fInfoFileMutex);
std::unique_lock lock(fInfoFileMutex);
// allowed state change:
// idle -> active
@@ -636,7 +636,7 @@ void RedistributeControl::updateState(uint32_t s)
void RedistributeControl::setEntryCount(uint32_t entryCount)
{
boost::mutex::scoped_lock lock(fInfoFileMutex);
std::unique_lock lock(fInfoFileMutex);
fRedistributeInfo.planned = entryCount;
rewind(fInfoFilePtr);
@@ -646,7 +646,7 @@ void RedistributeControl::setEntryCount(uint32_t entryCount)
void RedistributeControl::updateProgressInfo(uint32_t s, time_t t)
{
boost::mutex::scoped_lock lock(fInfoFileMutex);
std::unique_lock lock(fInfoFileMutex);
fRedistributeInfo.endTime = t;
switch (s)
@@ -665,7 +665,7 @@ void RedistributeControl::updateProgressInfo(uint32_t s, time_t t)
int RedistributeControl::handleJobMsg(messageqcpp::ByteStream& bs, messageqcpp::IOSocket& so)
{
// boost::mutex::scoped_lock jobLock(fJobMutex);
// std::unique_lock jobLock(fJobMutex);
uint32_t status = RED_TRANS_SUCCESS;

View File

@@ -83,8 +83,8 @@ class RedistributeControl
void logMessage(const std::string&);
boost::mutex fSessionMutex;
boost::mutex fInfoFileMutex;
std::mutex fSessionMutex;
std::mutex fInfoFileMutex;
boost::scoped_ptr<boost::thread> fControlThread;
boost::scoped_ptr<boost::thread> fWorkThread;

View File

@@ -65,13 +65,13 @@ using namespace execplan;
namespace redistribute
{
// static variables
boost::mutex RedistributeControlThread::fActionMutex;
std::mutex RedistributeControlThread::fActionMutex;
volatile bool RedistributeControlThread::fStopAction = false;
string RedistributeControlThread::fWesInUse;
void RedistributeControlThread::setStopAction(bool s)
{
boost::mutex::scoped_lock lock(fActionMutex);
std::unique_lock lock(fActionMutex);
fStopAction = s;
}
@@ -147,7 +147,7 @@ void RedistributeControlThread::doRedistribute()
fControl->logMessage(fErrorMsg + " @controlThread::doRedistribute");
{
boost::mutex::scoped_lock lock(fActionMutex);
std::unique_lock lock(fActionMutex);
fWesInUse.clear();
}
}
@@ -766,7 +766,7 @@ int RedistributeControlThread::connectToWes(int dbroot)
try
{
boost::mutex::scoped_lock lock(fActionMutex);
std::unique_lock lock(fActionMutex);
fWesInUse = oss.str();
fMsgQueueClient.reset(new MessageQueueClient(fWesInUse, fConfig));
}
@@ -783,7 +783,7 @@ int RedistributeControlThread::connectToWes(int dbroot)
if (ret != 0)
{
boost::mutex::scoped_lock lock(fActionMutex);
std::unique_lock lock(fActionMutex);
fWesInUse.clear();
fMsgQueueClient.reset();
@@ -797,7 +797,7 @@ void RedistributeControlThread::doStopAction()
fConfig = Config::makeConfig();
fControl = RedistributeControl::instance();
boost::mutex::scoped_lock lock(fActionMutex);
std::unique_lock lock(fActionMutex);
if (!fWesInUse.empty())
{

View File

@@ -118,7 +118,7 @@ class RedistributeControlThread
RedistributeControl* fControl;
static boost::mutex fActionMutex;
static std::mutex fActionMutex;
static volatile bool fStopAction;
static std::string fWesInUse;
};

View File

@@ -72,7 +72,7 @@ using namespace idbdatafile;
namespace redistribute
{
// static variables
boost::mutex RedistributeWorkerThread::fActionMutex;
std::mutex RedistributeWorkerThread::fActionMutex;
volatile bool RedistributeWorkerThread::fStopAction = false;
volatile bool RedistributeWorkerThread::fCommitted = false;
string RedistributeWorkerThread::fWesInUse;
@@ -85,7 +85,7 @@ RedistributeWorkerThread::RedistributeWorkerThread(ByteStream& bs, IOSocket& ios
RedistributeWorkerThread::~RedistributeWorkerThread()
{
boost::mutex::scoped_lock lock(fActionMutex);
std::unique_lock lock(fActionMutex);
if (fNewFilePtr)
closeFile(fNewFilePtr);
@@ -124,7 +124,7 @@ void RedistributeWorkerThread::handleRequest()
{
// clear stop flag if ever set.
{
boost::mutex::scoped_lock lock(fActionMutex);
std::unique_lock lock(fActionMutex);
fStopAction = false;
fCommitted = false;
}
@@ -172,7 +172,7 @@ void RedistributeWorkerThread::handleRequest()
sendResponse(RED_ACTN_REQUEST);
boost::mutex::scoped_lock lock(fActionMutex);
std::unique_lock lock(fActionMutex);
fWesInUse.clear();
fMsgQueueClient.reset();
@@ -865,7 +865,7 @@ int RedistributeWorkerThread::updateDbrm()
{
int rc1 = BRM::ERR_OK;
int rc2 = BRM::ERR_OK;
boost::mutex::scoped_lock lock(fActionMutex);
std::unique_lock lock(fActionMutex);
// cannot stop after extent map is updated.
if (!fStopAction)
@@ -1061,7 +1061,7 @@ void RedistributeWorkerThread::addToDirSet(const char* fileName, bool isSource)
void RedistributeWorkerThread::handleStop()
{
boost::mutex::scoped_lock lock(fActionMutex);
std::unique_lock lock(fActionMutex);
// cannot stop after extent map is updated.
if (!fCommitted)

View File

@@ -132,7 +132,7 @@ class RedistributeWorkerThread
// for segment file # workaround
// uint64_t fSegPerRoot;
static boost::mutex fActionMutex;
static std::mutex fActionMutex;
static volatile bool fStopAction;
static volatile bool fCommitted;
static std::string fWesInUse;

View File

@@ -32,7 +32,7 @@
#include <queue>
using namespace std;
#include <boost/thread/condition.hpp>
#include <condition_variable>
#include <boost/scoped_array.hpp>
#include <boost/thread.hpp>
using namespace boost;
@@ -89,7 +89,7 @@ void WECpiFeederThread::add2MsgQueue(ByteStream& Ibs)
// TODO creating copy is NOT good; later read from socket using a SBS
messageqcpp::SBS aSbs(new messageqcpp::ByteStream(Ibs));
Ibs.reset(); // forcefully clearing it
boost::mutex::scoped_lock aLock(fMsgQMutex);
std::unique_lock aLock(fMsgQMutex);
// cout << "pushing to the MsgQueue" << endl;
fMsgQueue.push(aSbs);
fFeederCond.notify_one(); // as per preference of Damon
@@ -102,11 +102,11 @@ void WECpiFeederThread::feedData2Cpi()
{
while (isContinue())
{
boost::mutex::scoped_lock aLock(fMsgQMutex);
std::unique_lock aLock(fMsgQMutex);
if (fMsgQueue.empty())
{
bool aTimedOut = fFeederCond.timed_wait(aLock, boost::posix_time::milliseconds(3000));
bool aTimedOut = (fFeederCond.wait_for(aLock, std::chrono::milliseconds(3000)) == std::cv_status::timeout);
if (!isContinue())
{
@@ -154,7 +154,7 @@ void WECpiFeederThread::feedData2Cpi()
bool WECpiFeederThread::isMsgQueueEmpty()
{
bool aRet = false;
boost::mutex::scoped_lock aLock(fMsgQMutex);
std::unique_lock aLock(fMsgQMutex);
aRet = fMsgQueue.empty();
aLock.unlock();
return aRet;
@@ -164,7 +164,7 @@ bool WECpiFeederThread::isMsgQueueEmpty()
void WECpiFeederThread::stopThread()
{
boost::mutex::scoped_lock aCondLock(fContMutex);
std::unique_lock aCondLock(fContMutex);
fContinue = false;
aCondLock.unlock();
fFeederCond.notify_all();
@@ -175,7 +175,7 @@ void WECpiFeederThread::stopThread()
bool WECpiFeederThread::isContinue()
{
boost::mutex::scoped_lock aCondLock(fContMutex);
std::unique_lock aCondLock(fContMutex);
return fContinue;
}

View File

@@ -77,14 +77,14 @@ class WECpiFeederThread
private:
WEDataLoader& fOwner;
boost::condition fFeederCond;
boost::mutex fMsgQMutex;
std::condition_variable fFeederCond;
std::mutex fMsgQMutex;
typedef std::queue<messageqcpp::SBS> WEMsgQueue;
WEMsgQueue fMsgQueue;
boost::thread* fpThread;
bool fContinue;
boost::mutex fContMutex;
std::mutex fContMutex;
// bool fPushing;
bool fStopped;

View File

@@ -38,7 +38,7 @@
#include <stdlib.h>
#include <time.h>
#include "bytestream.h"
#include "rwlock_local.h"
#include <iostream>
#include <fstream>
@@ -48,7 +48,7 @@
#include <map>
using namespace std;
#include <boost/thread/condition.hpp>
#include <condition_variable>
#include <boost/scoped_array.hpp>
#include <boost/thread.hpp>
#include <boost/filesystem.hpp>
@@ -569,7 +569,7 @@ void WEDataLoader::onCpimportSuccess()
if (aRet)
{
boost::mutex::scoped_lock aLock(fClntMsgMutex);
std::unique_lock aLock(fClntMsgMutex);
// aBrmRptParser.unserialize(obs); - was for testing
updateTxBytes(obs.length());
@@ -606,7 +606,7 @@ void WEDataLoader::onCpimportSuccess()
obs.reset();
obs << (ByteStream::byte)WE_CLT_SRV_CPIPASS;
obs << (ByteStream::byte)fPmId; // PM id
boost::mutex::scoped_lock aLock(fClntMsgMutex);
std::unique_lock aLock(fClntMsgMutex);
updateTxBytes(obs.length());
try
@@ -661,7 +661,7 @@ void WEDataLoader::onCpimportFailure()
if (aRet)
{
boost::mutex::scoped_lock aLock(fClntMsgMutex);
std::unique_lock aLock(fClntMsgMutex);
updateTxBytes(obs.length());
try
@@ -707,7 +707,7 @@ void WEDataLoader::sendCpimportFailureNotice()
ByteStream obs;
obs << (ByteStream::byte)WE_CLT_SRV_CPIFAIL;
obs << (ByteStream::byte)fPmId; // PM id
boost::mutex::scoped_lock aLock(fClntMsgMutex);
std::unique_lock aLock(fClntMsgMutex);
updateTxBytes(obs.length());
try
@@ -753,7 +753,7 @@ void WEDataLoader::onReceiveKeepAlive(ByteStream& Ibs)
ByteStream obs;
obs << (ByteStream::byte)WE_CLT_SRV_KEEPALIVE;
obs << (ByteStream::byte)fPmId; // PM id
boost::mutex::scoped_lock aLock(fClntMsgMutex);
std::unique_lock aLock(fClntMsgMutex);
updateTxBytes(obs.length());
try
@@ -812,7 +812,7 @@ void WEDataLoader::onReceiveKeepAlive(ByteStream& Ibs)
ByteStream obs;
obs << (ByteStream::byte)WE_CLT_SRV_EOD;
obs << (ByteStream::byte)fPmId; // PM id
boost::mutex::scoped_lock aLock(fClntMsgMutex);
std::unique_lock aLock(fClntMsgMutex);
updateTxBytes(obs.length());
try
@@ -934,7 +934,7 @@ void WEDataLoader::onReceiveEod(ByteStream& Ibs)
ByteStream obs;
obs << (ByteStream::byte)WE_CLT_SRV_EOD;
obs << (ByteStream::byte)fPmId; // PM id
boost::mutex::scoped_lock aLock(fClntMsgMutex);
std::unique_lock aLock(fClntMsgMutex);
updateTxBytes(obs.length());
try
@@ -1033,7 +1033,7 @@ void WEDataLoader::onReceiveMode(ByteStream& Ibs)
aObs << (ByteStream::byte)WE_CLT_SRV_DBRCNT;
aObs << (ByteStream::byte)fPmId;
aObs << (ByteStream::byte)aDbCnt;
boost::mutex::scoped_lock aLock(fClntMsgMutex);
std::unique_lock aLock(fClntMsgMutex);
updateTxBytes(aObs.length());
try
@@ -1124,7 +1124,7 @@ void WEDataLoader::onReceiveCmdLineArgs(ByteStream& Ibs)
}
obs << (ByteStream::byte)fPmId; // PM id
boost::mutex::scoped_lock aLock(fClntMsgMutex);
std::unique_lock aLock(fClntMsgMutex);
updateTxBytes(obs.length());
try
@@ -1259,7 +1259,7 @@ void WEDataLoader::onReceiveCleanup(ByteStream& Ibs)
else
obs << (ByteStream::byte)0; // cleanup failed
boost::mutex::scoped_lock aLock(fClntMsgMutex);
std::unique_lock aLock(fClntMsgMutex);
updateTxBytes(obs.length());
try
@@ -1310,7 +1310,7 @@ void WEDataLoader::onReceiveRollback(ByteStream& Ibs)
else
obs << (ByteStream::byte)0; // Rollback failed
boost::mutex::scoped_lock aLock(fClntMsgMutex);
std::unique_lock aLock(fClntMsgMutex);
updateTxBytes(obs.length());
try
@@ -1353,7 +1353,7 @@ void WEDataLoader::onReceiveImportFileName(ByteStream& Ibs)
obs << (ByteStream::byte)WE_CLT_SRV_IMPFILEERROR;
obs << (ByteStream::byte)fPmId;
updateTxBytes(obs.length());
boost::mutex::scoped_lock aLock(fClntMsgMutex);
std::unique_lock aLock(fClntMsgMutex);
try
{
@@ -1390,7 +1390,7 @@ void WEDataLoader::onReceiveImportFileName(ByteStream& Ibs)
ByteStream obs;
obs << (ByteStream::byte)WE_CLT_SRV_IMPFILEERROR;
obs << (ByteStream::byte)fPmId; // PM id
boost::mutex::scoped_lock aLock(fClntMsgMutex);
std::unique_lock aLock(fClntMsgMutex);
updateTxBytes(obs.length());
try
@@ -1470,7 +1470,7 @@ void WEDataLoader::onReceiveErrFileRqst(ByteStream& Ibs)
if (aRet)
{
boost::mutex::scoped_lock aLock(fClntMsgMutex);
std::unique_lock aLock(fClntMsgMutex);
updateTxBytes(obs.length());
try
@@ -1517,7 +1517,7 @@ void WEDataLoader::onReceiveBadFileRqst(ByteStream& Ibs)
if (aRet)
{
boost::mutex::scoped_lock aLock(fClntMsgMutex);
std::unique_lock aLock(fClntMsgMutex);
updateTxBytes(obs.length());
try
@@ -1569,7 +1569,7 @@ void WEDataLoader::sendDataRequest()
}
}
boost::mutex::scoped_lock aLock(fClntMsgMutex);
std::unique_lock aLock(fClntMsgMutex);
ByteStream obs;
obs << (ByteStream::byte)WE_CLT_SRV_DATARQST;
obs << (ByteStream::byte)fPmId; // PM id

View File

@@ -30,7 +30,7 @@
#pragma once
#include "rwlock_local.h"
#include "resourcemanager.h"
#include "we_simplesyslog.h"
@@ -189,7 +189,7 @@ class WEDataLoader : public Observer
// CPI Feeder Thread
WECpiFeederThread* fpCfThread;
boost::mutex fClntMsgMutex; // mutex in sending messages to client.
std::mutex fClntMsgMutex; // mutex in sending messages to client.
// static bool fTearDownCpimport; // @bug 4267
bool fTearDownCpimport; // @bug 4267

View File

@@ -295,8 +295,8 @@ uint8_t WE_DDLCommandProc::writeSystable(ByteStream& bs, std::string& err)
{
// MCOL-66 The DBRM can't handle concurrent transactions to sys tables
// TODO: This may be redundant
static boost::mutex dbrmMutex;
boost::mutex::scoped_lock lk(dbrmMutex);
static std::mutex dbrmMutex;
std::unique_lock lk(dbrmMutex);
error = fWEWrapper.insertColumnRec_SYS(txnID, cscColTypeList, colStructs, colValuesList,
dctnryStructList, dctnryValueList, SYSCOLUMN_BASE);

View File

@@ -64,7 +64,7 @@ typedef std::vector<FileInfo> Files;
typedef std::map<uint32_t, Files> columnMap;
typedef std::map<int, columnMap*> allColumnMap;
allColumnMap wholeMap;
boost::mutex columnMapLock;
std::mutex columnMapLock;
ActiveThreadCounter* activeThreadCounter;
size_t readFillBuffer(idbdatafile::IDBDataFile* pFile, char* buffer, size_t bytesReq)
@@ -243,7 +243,7 @@ struct ColumnThread
}
}
boost::mutex::scoped_lock lk(columnMapLock);
std::unique_lock lk(columnMapLock);
// cout << "Current size of columnsMap is " << columnsMap.size() << endl;
allColumnMap::iterator colMapiter = wholeMap.find(fKey);
@@ -330,7 +330,7 @@ int WE_GetFileSizes::processTable(messageqcpp::ByteStream& bs, std::string& errM
columnMap* columnsMap = new columnMap();
{
boost::mutex::scoped_lock lk(columnMapLock);
std::unique_lock lk(columnMapLock);
wholeMap[key] = columnsMap;
}
@@ -378,7 +378,7 @@ int WE_GetFileSizes::processTable(messageqcpp::ByteStream& bs, std::string& errM
// Build the message to send to the caller
bs.reset();
boost::mutex::scoped_lock lk(columnMapLock);
std::unique_lock lk(columnMapLock);
allColumnMap::iterator colMapiter = wholeMap.find(key);
if (colMapiter != wholeMap.end())

View File

@@ -27,7 +27,8 @@
* Author: bpaul@calpont.com
*/
#include <boost/thread/mutex.hpp>
#include <map>
#include <mutex>
#include "we_observer.h"
@@ -62,7 +63,6 @@ Subject::~Subject()
void Subject::attach(Observer* Obs)
{
boost::mutex::scoped_lock aLstLock;
fObs.push_back(Obs);
}
@@ -70,7 +70,6 @@ void Subject::attach(Observer* Obs)
void Subject::detach(Observer* Obs)
{
boost::mutex::scoped_lock aLstLock;
Observers::iterator aIt = fObs.begin();
while (aIt != fObs.end())
@@ -87,7 +86,6 @@ void Subject::detach(Observer* Obs)
void Subject::notify()
{
boost::mutex::scoped_lock aLstLock;
Observers::iterator aIt = fObs.begin();
while (aIt != fObs.end())

View File

@@ -30,6 +30,7 @@
#pragma once
#include <list>
#include <mutex>
namespace WriteEngine
{

View File

@@ -29,7 +29,8 @@
#include <algorithm>
#include <unistd.h>
using namespace std;
#include <boost/thread/mutex.hpp>
#include <map>
#include <mutex>
#include <boost/scoped_ptr.hpp>
using namespace boost;
@@ -58,13 +59,13 @@ namespace WriteEngine
{
BRMWrapper* volatile BRMWrapper::m_instance = NULL;
boost::thread_specific_ptr<int> BRMWrapper::m_ThreadDataPtr;
boost::mutex BRMWrapper::m_instanceCreateMutex;
std::mutex BRMWrapper::m_instanceCreateMutex;
bool BRMWrapper::m_useVb = true;
OID BRMWrapper::m_curVBOid = INVALID_NUM;
IDBDataFile* BRMWrapper::m_curVBFile = NULL;
boost::mutex vbFileLock;
std::mutex vbFileLock;
struct fileInfoCompare // lt operator
{
bool operator()(const File& lhs, const File& rhs) const
@@ -300,7 +301,7 @@ BRMWrapper* BRMWrapper::getInstance()
{
if (m_instance == 0)
{
boost::mutex::scoped_lock lock(m_instanceCreateMutex);
std::unique_lock lock(m_instanceCreateMutex);
if (m_instance == 0)
{
@@ -1638,7 +1639,7 @@ int BRMWrapper::writeVB(IDBDataFile* pSourceFile, const VER_t transID, const OID
fileInfo.fSegment = 0;
// fileInfo.fDbRoot = (freeList[0].vbOID % rootCnt) + 1;
fileInfo.fDbRoot = dbRoot;
boost::mutex::scoped_lock lk(vbFileLock);
std::unique_lock lk(vbFileLock);
pTargetFile = openFile(fileInfo, "r+b", true);
if (pTargetFile == NULL)

View File

@@ -464,7 +464,7 @@ class BRMWrapper : public WEObj
static BRMWrapper* volatile m_instance;
static boost::thread_specific_ptr<int> m_ThreadDataPtr;
static boost::mutex m_instanceCreateMutex;
static std::mutex m_instanceCreateMutex;
EXPORT static bool m_useVb;

View File

@@ -59,9 +59,9 @@ string Config::m_bulkRoot;
unsigned long Config::fDBRootChangeCount = 0;
time_t Config::fCacheTime = 0;
boost::mutex Config::fCacheLock;
std::mutex Config::fCacheLock;
#ifdef SHARED_NOTHING_DEMO_2
boost::mutex Config::m_bulkRoot_lk;
std::mutex Config::m_bulkRoot_lk;
#endif
int Config::m_WaitPeriod = DEFAULT_WAIT_PERIOD;
unsigned Config::m_FilesPerColumnPartition = DEFAULT_FILES_PER_COLUMN_PARTITION;
@@ -86,7 +86,7 @@ string Config::m_VersionBufferDir;
******************************************************************************/
void Config::initConfigCache()
{
boost::mutex::scoped_lock lk(fCacheLock);
std::unique_lock lk(fCacheLock);
checkReload();
}
@@ -330,7 +330,7 @@ void Config::checkReload()
******************************************************************************/
size_t Config::DBRootCount()
{
boost::mutex::scoped_lock lk(fCacheLock);
std::unique_lock lk(fCacheLock);
checkReload();
return m_dbRootCount;
@@ -346,7 +346,7 @@ size_t Config::DBRootCount()
******************************************************************************/
std::string Config::getDBRootByIdx(unsigned idx)
{
boost::mutex::scoped_lock lk(fCacheLock);
std::unique_lock lk(fCacheLock);
checkReload();
if (idx >= m_dbRootPath.size())
@@ -368,7 +368,7 @@ std::string Config::getDBRootByIdx(unsigned idx)
******************************************************************************/
void Config::getDBRootPathList(std::vector<std::string>& dbRootPathList)
{
boost::mutex::scoped_lock lk(fCacheLock);
std::unique_lock lk(fCacheLock);
checkReload();
dbRootPathList.clear();
@@ -385,7 +385,7 @@ void Config::getDBRootPathList(std::vector<std::string>& dbRootPathList)
******************************************************************************/
std::string Config::getDBRootByNum(unsigned num)
{
boost::mutex::scoped_lock lk(fCacheLock);
std::unique_lock lk(fCacheLock);
checkReload();
Config::intstrmap_t::const_iterator iter = m_dbRootPathMap.find(num);
@@ -409,7 +409,7 @@ std::string Config::getDBRootByNum(unsigned num)
******************************************************************************/
void Config::getRootIdList(std::vector<uint16_t>& rootIds)
{
boost::mutex::scoped_lock lk(fCacheLock);
std::unique_lock lk(fCacheLock);
checkReload();
rootIds = m_dbRootId;
@@ -425,7 +425,7 @@ void Config::getRootIdList(std::vector<uint16_t>& rootIds)
******************************************************************************/
std::string Config::getBulkRoot()
{
boost::mutex::scoped_lock lk(fCacheLock);
std::unique_lock lk(fCacheLock);
checkReload();
return m_bulkRoot;
@@ -435,7 +435,7 @@ std::string Config::getBulkRoot()
void Config::getSharedNothingRoot(char* ret)
{
string root;
boost::mutex::scoped_lock lk(m_bulkRoot_lk);
std::unique_lock lk(m_bulkRoot_lk);
root = config::Config::makeConfig()->getConfig("WriteEngine", "SharedNothingRoot");
strncpy(ret, root.c_str(), FILE_NAME_SIZE);
@@ -452,7 +452,7 @@ void Config::getSharedNothingRoot(char* ret)
******************************************************************************/
int Config::getWaitPeriod()
{
boost::mutex::scoped_lock lk(fCacheLock);
std::unique_lock lk(fCacheLock);
checkReload();
return m_WaitPeriod;
@@ -468,7 +468,7 @@ int Config::getWaitPeriod()
******************************************************************************/
unsigned Config::getFilesPerColumnPartition()
{
boost::mutex::scoped_lock lk(fCacheLock);
std::unique_lock lk(fCacheLock);
checkReload();
return m_FilesPerColumnPartition;
@@ -484,7 +484,7 @@ unsigned Config::getFilesPerColumnPartition()
******************************************************************************/
unsigned Config::getExtentsPerSegmentFile()
{
boost::mutex::scoped_lock lk(fCacheLock);
std::unique_lock lk(fCacheLock);
checkReload();
return m_ExtentsPerSegmentFile;
@@ -507,7 +507,7 @@ unsigned Config::getExtentsPerSegmentFile()
******************************************************************************/
int Config::getBulkProcessPriority()
{
boost::mutex::scoped_lock lk(fCacheLock);
std::unique_lock lk(fCacheLock);
checkReload();
return m_BulkProcessPriority;
@@ -521,7 +521,7 @@ int Config::getBulkProcessPriority()
******************************************************************************/
std::string Config::getBulkRollbackDir()
{
boost::mutex::scoped_lock lk(fCacheLock);
std::unique_lock lk(fCacheLock);
checkReload();
return m_BulkRollbackDir;
@@ -535,7 +535,7 @@ std::string Config::getBulkRollbackDir()
******************************************************************************/
bool Config::getFastDelete()
{
boost::mutex::scoped_lock lk(fCacheLock);
std::unique_lock lk(fCacheLock);
checkReload();
return m_FastDelete;
@@ -549,7 +549,7 @@ bool Config::getFastDelete()
******************************************************************************/
unsigned Config::getMaxFileSystemDiskUsage()
{
boost::mutex::scoped_lock lk(fCacheLock);
std::unique_lock lk(fCacheLock);
checkReload();
return m_MaxFileSystemDiskUsage;
@@ -564,7 +564,7 @@ unsigned Config::getMaxFileSystemDiskUsage()
******************************************************************************/
unsigned Config::getNumCompressedPadBlks()
{
boost::mutex::scoped_lock lk(fCacheLock);
std::unique_lock lk(fCacheLock);
checkReload();
return m_NumCompressedPadBlks;
@@ -578,7 +578,7 @@ unsigned Config::getNumCompressedPadBlks()
******************************************************************************/
bool Config::getParentOAMModuleFlag()
{
boost::mutex::scoped_lock lk(fCacheLock);
std::unique_lock lk(fCacheLock);
checkReload();
return m_ParentOAMModuleFlag;
@@ -592,7 +592,7 @@ bool Config::getParentOAMModuleFlag()
******************************************************************************/
std::string Config::getLocalModuleType()
{
boost::mutex::scoped_lock lk(fCacheLock);
std::unique_lock lk(fCacheLock);
checkReload();
return m_LocalModuleType;
@@ -606,7 +606,7 @@ std::string Config::getLocalModuleType()
******************************************************************************/
uint16_t Config::getLocalModuleID()
{
boost::mutex::scoped_lock lk(fCacheLock);
std::unique_lock lk(fCacheLock);
checkReload();
return m_LocalModuleID;
@@ -622,7 +622,7 @@ uint16_t Config::getLocalModuleID()
******************************************************************************/
std::string Config::getVBRoot()
{
boost::mutex::scoped_lock lk(fCacheLock);
std::unique_lock lk(fCacheLock);
checkReload();
return m_VersionBufferDir;
@@ -639,7 +639,7 @@ std::string Config::getVBRoot()
******************************************************************************/
bool Config::hasLocalDBRootListChanged()
{
boost::mutex::scoped_lock lk(fCacheLock);
std::unique_lock lk(fCacheLock);
if (fDBRootChangeCount > 0)
{

View File

@@ -182,9 +182,9 @@ class Config
static std::string m_bulkRoot; // root path for bulk operation
static unsigned long fDBRootChangeCount; // track recent DBRoot changes
static time_t fCacheTime; // timestamp associated w/cache
static boost::mutex fCacheLock; // mutex for m_dbRoot sync
static std::mutex fCacheLock; // mutex for m_dbRoot sync
#ifdef SHARED_NOTHING_DEMO_2
static boost::mutex m_bulkRoot_lk; // mutex for m_bulkRoot sync
static std::mutex m_bulkRoot_lk; // mutex for m_bulkRoot sync
#endif
static int m_WaitPeriod; // secs to wait for transaction
static unsigned m_FilesPerColumnPartition; //# seg files per partition

View File

@@ -474,7 +474,7 @@ void DBRootExtentTracker::logFirstDBRootSelection() const
bool DBRootExtentTracker::nextSegFile(uint16_t& dbRoot, uint32_t& partition, uint16_t& segment, HWM& localHwm,
BRM::LBID_t& startLbid)
{
boost::mutex::scoped_lock lock(fDBRootExtTrkMutex);
std::unique_lock lock(fDBRootExtTrkMutex);
fCurrentDBRootIdx++;
@@ -515,7 +515,7 @@ bool DBRootExtentTracker::nextSegFile(uint16_t& dbRoot, uint32_t& partition, uin
const std::vector<DBRootExtentInfo>& DBRootExtentTracker::getDBRootExtentList()
{
boost::mutex::scoped_lock lock(fDBRootExtTrkMutex);
std::unique_lock lock(fDBRootExtTrkMutex);
return fDBRootExtentList;
}

View File

@@ -34,7 +34,8 @@
#pragma once
#include <boost/thread/mutex.hpp>
#include <map>
#include <mutex>
#include <vector>
#include "we_type.h"
@@ -184,7 +185,7 @@ class DBRootExtentTracker
*/
inline int getCurrentDBRootIdx()
{
boost::mutex::scoped_lock lock(fDBRootExtTrkMutex);
std::unique_lock lock(fDBRootExtTrkMutex);
return fCurrentDBRootIdx;
}
@@ -199,7 +200,7 @@ class DBRootExtentTracker
OID fOID; // applicable colunn OID
long long fBlksPerExtent; // blocks per extent for fOID
Log* fLog; // logger
boost::mutex fDBRootExtTrkMutex; // mutex to access fDBRootExtentList
std::mutex fDBRootExtTrkMutex; // mutex to access fDBRootExtentList
int fCurrentDBRootIdx; // Index into fDBRootExtentList,
// DBRoot where current extent is
// being added

View File

@@ -58,9 +58,9 @@ using namespace idbdatafile;
namespace WriteEngine
{
/*static*/ boost::mutex FileOp::m_createDbRootMutexes;
/*static*/ boost::mutex FileOp::m_mkdirMutex;
/*static*/ std::map<int, boost::mutex> FileOp::m_DbRootAddExtentMutexes;
/*static*/ std::mutex FileOp::m_createDbRootMutexes;
/*static*/ std::mutex FileOp::m_mkdirMutex;
/*static*/ std::map<int, std::mutex> FileOp::m_DbRootAddExtentMutexes;
// in 1 call to fwrite(), during initialization
// StopWatch timer;
@@ -116,7 +116,7 @@ void FileOp::closeFile(IDBDataFile* pFile) const
***********************************************************/
int FileOp::createDir(const char* dirName, mode_t mode) const
{
boost::mutex::scoped_lock lk(m_mkdirMutex);
std::unique_lock lk(m_mkdirMutex);
int rc = IDBPolicy::mkdir(dirName);
if (rc != 0)
@@ -993,7 +993,7 @@ int FileOp::initColumnExtent(IDBDataFile* pFile, uint16_t dbRoot, int nBlocks, c
Stats::startParseEvent(WE_STATS_WAIT_TO_CREATE_COL_EXTENT);
#endif
boost::mutex::scoped_lock lk(m_DbRootAddExtentMutexes[dbRoot]);
std::unique_lock lk(m_DbRootAddExtentMutexes[dbRoot]);
#ifdef PROFILE
if (bExpandExtent)
@@ -1714,7 +1714,7 @@ int FileOp::initDctnryExtent(IDBDataFile* pFile, uint16_t dbRoot, int nBlocks, u
Stats::startParseEvent(WE_STATS_WAIT_TO_CREATE_DCT_EXTENT);
#endif
boost::mutex::scoped_lock lk(m_DbRootAddExtentMutexes[dbRoot]);
std::unique_lock lk(m_DbRootAddExtentMutexes[dbRoot]);
#ifdef PROFILE
if (bExpandExtent)
@@ -1800,7 +1800,7 @@ int FileOp::initDctnryExtent(IDBDataFile* pFile, uint16_t dbRoot, int nBlocks, u
/* static */
void FileOp::initDbRootExtentMutexes()
{
boost::mutex::scoped_lock lk(m_createDbRootMutexes);
std::unique_lock lk(m_createDbRootMutexes);
if (m_DbRootAddExtentMutexes.size() == 0)
{

View File

@@ -453,13 +453,13 @@ class FileOp : public BlockOp, public WeUIDGID
bool m_isFix;
// protect creation of m_DbRootAddExtentMutexes
static boost::mutex m_createDbRootMutexes;
static std::mutex m_createDbRootMutexes;
// Mutexes used to serialize extent creation within each DBRoot
static std::map<int, boost::mutex> m_DbRootAddExtentMutexes;
static std::map<int, std::mutex> m_DbRootAddExtentMutexes;
// protect race condition in creating directories
static boost::mutex m_mkdirMutex;
static std::mutex m_mkdirMutex;
char* m_buffer; // buffer used with setvbuf()
};

View File

@@ -108,7 +108,7 @@ void Log::logMsg(const char* msg, int code, MsgLevel level)
{
// log to log file and error log file within scope of mutex lock.
// logSyslog uses SimpleSyslog which has it's own lock.
boost::mutex::scoped_lock lk(m_WriteLockMutex);
std::unique_lock lk(m_WriteLockMutex);
m_errLogFile << oss.str() << std::endl;
m_logFile << oss.str() << std::endl;
@@ -128,7 +128,7 @@ void Log::logMsg(const char* msg, int code, MsgLevel level)
if ((level != MSGLVL_INFO2) || (m_bConsoleOutput))
formatMsg(msg, level, oss2);
boost::mutex::scoped_lock lk(m_WriteLockMutex);
std::unique_lock lk(m_WriteLockMutex);
m_logFile << oss.str() << std::endl;

View File

@@ -99,7 +99,7 @@ class Log : public WEObj
std::ofstream m_logFile; // log file stream
std::ofstream m_errLogFile; // error log file stream
boost::mutex m_WriteLockMutex; // logging mutex
std::mutex m_WriteLockMutex; // logging mutex
};
} // namespace WriteEngine

View File

@@ -663,7 +663,7 @@ bool RBMetaWriter::backupDctnryHWMChunk(OID dctnryOID, uint16_t dbRoot, uint32_t
{
// Use scoped lock to perform "find"
boost::mutex::scoped_lock lock(fRBChunkDctnryMutex);
std::unique_lock lock(fRBChunkDctnryMutex);
if ((fLog) && (fLog->isDebug(DEBUG_1)))
printDctnryChunkList(chunkInfo, "when searching ");
@@ -705,7 +705,7 @@ bool RBMetaWriter::backupDctnryHWMChunk(OID dctnryOID, uint16_t dbRoot, uint32_t
{
// Use scoped lock to perform "erase"
boost::mutex::scoped_lock lock(fRBChunkDctnryMutex);
std::unique_lock lock(fRBChunkDctnryMutex);
fRBChunkDctnrySet.erase(chunkInfoFound);
if ((fLog) && (fLog->isDebug(DEBUG_1)))

View File

@@ -34,7 +34,8 @@
#include <set>
#include <map>
#include <vector>
#include <boost/thread/mutex.hpp>
#include <map>
#include <mutex>
#include "we_type.h"
#include "brmtypes.h"
@@ -314,7 +315,7 @@ class RBMetaWriter : public WeUIDGID
Log* fLog; // import log file
bool fCreatedSubDir; // has subdir path been created
RBChunkSet fRBChunkDctnrySet; // Dctnry HWM chunk info
boost::mutex fRBChunkDctnryMutex; // Mutex lock for RBChunkSet
std::mutex fRBChunkDctnryMutex; // Mutex lock for RBChunkSet
OID fTableOID; // OID of relevant table
std::string fTableName; // Name of relevant table
};

View File

@@ -69,7 +69,7 @@ void SimpleSysLog::logMsg(const logging::Message::Args& msgArgs, logging::LOG_TY
logging::Message m(msgId);
m.format(msgArgs);
boost::mutex::scoped_lock lk(fWriteLockMutex);
std::unique_lock lk(fWriteLockMutex);
switch (logType)
{

View File

@@ -66,7 +66,7 @@ class SimpleSysLog
static SimpleSysLog* fSysLogger;
logging::LoggingID fLoggingID;
boost::mutex fWriteLockMutex; // logging mutex
std::mutex fWriteLockMutex; // logging mutex
};
#undef EXPORT

View File

@@ -29,8 +29,8 @@ namespace WriteEngine
{
#ifdef PROFILE
/* static */ bool Stats::fProfiling = false;
/* static */ boost::mutex Stats::fRegisterReaderMutex;
/* static */ boost::mutex Stats::fRegisterParseMutex;
/* static */ std::mutex Stats::fRegisterReaderMutex;
/* static */ std::mutex Stats::fRegisterParseMutex;
/* static */ std::vector<pthread_t> Stats::fReadProfThreads;
/* static */ std::vector<pthread_t> Stats::fParseProfThreads;
/* static */ std::vector<logging::StopWatch> Stats::fReadStopWatch;
@@ -111,7 +111,7 @@ void Stats::enableProfiling(int nReadThreads, int nParseThreads)
***********************************************************/
void Stats::registerReadProfThread()
{
boost::mutex::scoped_lock lk(fRegisterReaderMutex);
std::unique_lock lk(fRegisterReaderMutex);
fReadProfThreads.push_back(pthread_self());
logging::StopWatch readStopWatch;
@@ -128,7 +128,7 @@ void Stats::registerReadProfThread()
***********************************************************/
void Stats::registerParseProfThread()
{
boost::mutex::scoped_lock lk(fRegisterParseMutex);
std::unique_lock lk(fRegisterParseMutex);
fParseProfThreads.push_back(pthread_self());
logging::StopWatch parseStopWatch;

View File

@@ -24,7 +24,8 @@
#pragma once
#include <we_obj.h>
#ifdef PROFILE
#include <boost/thread/mutex.hpp>
#include <map>
#include <mutex>
#include "stopwatch.h"
#endif
@@ -159,10 +160,10 @@ class Stats
static bool fProfiling; // Is profiling enabled
// Protect concurrent addition of Readers
static boost::mutex fRegisterReaderMutex;
static std::mutex fRegisterReaderMutex;
// Protect concurrent addition of Parsers
static boost::mutex fRegisterParseMutex;
static std::mutex fRegisterParseMutex;
// Read threads to be profiled
static std::vector<pthread_t> fReadProfThreads;

View File

@@ -31,7 +31,7 @@
#include "we_sdhandler.h"
#include "we_splitterapp.h"
#include <boost/thread/condition.hpp>
#include <condition_variable>
#include <boost/scoped_array.hpp>
#include <boost/thread.hpp>
using namespace boost;
@@ -279,7 +279,7 @@ void WEFileReadThread::add2InputDataFileList(std::string& FileName)
void WEFileReadThread::shutdown()
{
this->setContinue(false);
boost::mutex::scoped_lock aLock(fFileMutex); // wait till readDataFile() finish
std::unique_lock aLock(fFileMutex); // wait till readDataFile() finish
// if(fInFile.is_open()) fInFile.close(); //@BUG 4326
if (fIfFile.is_open())
@@ -323,7 +323,7 @@ void WEFileReadThread::feedData()
// cout << "Length " << aSbs->length() <<endl; - for debug
fSdh.updateRowTx(aRowCnt, TgtPmId);
boost::mutex::scoped_lock aLock(fSdh.fSendMutex);
std::unique_lock aLock(fSdh.fSendMutex);
fSdh.send2Pm(aSbs, TgtPmId);
aLock.unlock();
setTgtPmId(0); // reset PmId. Send the data to next least data
@@ -367,7 +367,7 @@ void WEFileReadThread::feedData()
//------------------------------------------------------------------------------
unsigned int WEFileReadThread::readDataFile(messageqcpp::SBS& Sbs)
{
boost::mutex::scoped_lock aLock(fFileMutex);
std::unique_lock aLock(fFileMutex);
// For now we are going to send KEEPALIVES
//*Sbs << (ByteStream::byte)(WE_CLT_SRV_KEEPALIVE);
@@ -431,7 +431,7 @@ unsigned int WEFileReadThread::readDataFile(messageqcpp::SBS& Sbs)
//------------------------------------------------------------------------------
unsigned int WEFileReadThread::readBinaryDataFile(messageqcpp::SBS& Sbs, unsigned int recLen)
{
boost::mutex::scoped_lock aLock(fFileMutex);
std::unique_lock aLock(fFileMutex);
if ((fInFile.good()) && (!fInFile.eof()))
{

View File

@@ -137,7 +137,7 @@ class WEFileReadThread
WESDHandler& fSdh;
boost::thread* fpThread;
boost::mutex fFileMutex;
std::mutex fFileMutex;
bool fContinue;
std::string fInFileName;
std::istream fInFile; //@BUG 4326

View File

@@ -27,7 +27,7 @@
* Author: bpaul
*/
#include <boost/thread/condition.hpp>
#include <condition_variable>
#include <boost/scoped_array.hpp>
#include <boost/thread.hpp>
using namespace boost;

View File

@@ -38,7 +38,7 @@ using namespace std;
#include <sys/time.h>
#include <boost/thread/condition.hpp>
#include <condition_variable>
#include <boost/scoped_array.hpp>
#include <boost/thread.hpp>
using namespace boost;
@@ -82,7 +82,7 @@ namespace WriteEngine
void WEPmList::addPm2List(int PmId)
{
boost::mutex::scoped_lock aLock(fListMutex);
std::unique_lock aLock(fListMutex);
fPmList.push_back(PmId);
aLock.unlock();
}
@@ -91,7 +91,7 @@ void WEPmList::addPm2List(int PmId)
void WEPmList::addPriorityPm2List(int PmId)
{
boost::mutex::scoped_lock aLock(fListMutex);
std::unique_lock aLock(fListMutex);
fPmList.push_front(PmId);
aLock.unlock();
}
@@ -99,7 +99,7 @@ void WEPmList::addPriorityPm2List(int PmId)
int WEPmList::getNextPm()
{
boost::mutex::scoped_lock aLock(fListMutex);
std::unique_lock aLock(fListMutex);
int aPmId = 0;
if (!fPmList.empty())
@@ -116,7 +116,7 @@ int WEPmList::getNextPm()
// so that the sendingthreads will not keep sending data.
void WEPmList::clearPmList()
{
boost::mutex::scoped_lock aLock(fListMutex);
std::unique_lock aLock(fListMutex);
if (!fPmList.empty())
fPmList.clear();
@@ -128,7 +128,7 @@ void WEPmList::clearPmList()
bool WEPmList::check4Pm(int PmId)
{
boost::mutex::scoped_lock aLock(fListMutex);
std::unique_lock aLock(fListMutex);
WePmList::iterator aIt = fPmList.begin();
bool aFound = false;
@@ -273,7 +273,7 @@ void WESDHandler::send2Pm(ByteStream& Bs, unsigned int PmId)
{
if (fWeSplClients[aIdx] != 0)
{
boost::mutex::scoped_lock aLock(fWeSplClients[aIdx]->fWriteMutex);
std::unique_lock aLock(fWeSplClients[aIdx]->fWriteMutex);
fWeSplClients[aIdx]->write(Bs);
aLock.unlock();
}
@@ -281,7 +281,7 @@ void WESDHandler::send2Pm(ByteStream& Bs, unsigned int PmId)
}
else
{
boost::mutex::scoped_lock aLock(fWeSplClients[PmId]->fWriteMutex);
std::unique_lock aLock(fWeSplClients[PmId]->fWriteMutex);
fWeSplClients[PmId]->write(Bs);
aLock.unlock();
}
@@ -302,7 +302,7 @@ void WESDHandler::send2Pm(messageqcpp::SBS& Sbs, unsigned int PmId)
{
if (fWeSplClients[aIdx] != 0)
{
boost::mutex::scoped_lock aLock(fWeSplClients[aIdx]->fSentQMutex);
std::unique_lock aLock(fWeSplClients[aIdx]->fSentQMutex);
fWeSplClients[aIdx]->add2SendQueue(Sbs);
aLock.unlock();
}
@@ -310,7 +310,7 @@ void WESDHandler::send2Pm(messageqcpp::SBS& Sbs, unsigned int PmId)
}
else
{
boost::mutex::scoped_lock aLock(fWeSplClients[PmId]->fSentQMutex);
std::unique_lock aLock(fWeSplClients[PmId]->fSentQMutex);
fWeSplClients[PmId]->add2SendQueue(Sbs);
aLock.unlock();
}
@@ -353,7 +353,7 @@ void WESDHandler::checkForRespMsgs()
while (isContinue())
{
boost::mutex::scoped_lock aLock(fRespMutex);
std::unique_lock aLock(fRespMutex);
// NOTE - if isContinue is not checked thread will hang on shutdown
// by locking again on fRespList.empty()
@@ -418,7 +418,7 @@ void WESDHandler::checkForRespMsgs()
void WESDHandler::add2RespQueue(const messageqcpp::SBS& Sbs)
{
boost::mutex::scoped_lock aLock(fRespMutex);
std::unique_lock aLock(fRespMutex);
fRespList.push_back(Sbs);
aLock.unlock();
// cout <<"Notifing from add2RespQueue" << endl;
@@ -833,7 +833,7 @@ void WESDHandler::cancelOutstandingCpimports()
if (fWeSplClients[aCnt]->isConnected())
{
boost::mutex::scoped_lock aLock(fWeSplClients[aCnt]->fWriteMutex);
std::unique_lock aLock(fWeSplClients[aCnt]->fWriteMutex);
fWeSplClients[aCnt]->write(aBs);
aLock.unlock();
}
@@ -988,7 +988,7 @@ void WESDHandler::shutdown()
}
}
boost::mutex::scoped_lock aLock(fRespMutex);
std::unique_lock aLock(fRespMutex);
this->setContinue(false);
usleep(100000); // so that response thread get updated.
fRespCond.notify_all();
@@ -1935,7 +1935,7 @@ void WESDHandler::doRollback()
aBs << (ByteStream::quadbyte)fTableOId;
aBs << fRef.fCmdArgs.getTableName();
aBs << aAppName;
boost::mutex::scoped_lock aLock(fSendMutex);
std::unique_lock aLock(fSendMutex);
send2Pm(aBs);
aLock.unlock();
}
@@ -1951,7 +1951,7 @@ void WESDHandler::doCleanup(bool deleteHdfsTempDbFiles)
aBs << (ByteStream::byte)WE_CLT_SRV_CLEANUP;
aBs << (ByteStream::quadbyte)fTableOId;
aBs << (ByteStream::byte)deleteHdfsTempDbFiles;
boost::mutex::scoped_lock aLock(fSendMutex);
std::unique_lock aLock(fSendMutex);
send2Pm(aBs);
aLock.unlock();
}

View File

@@ -70,7 +70,7 @@ class WEPmList
private:
typedef std::list<int> WePmList; // List to add in front
WePmList fPmList;
boost::mutex fListMutex; // mutex controls add/remove
std::mutex fListMutex; // mutex controls add/remove
};
//------------------------------------------------------------------------------
@@ -262,10 +262,10 @@ class WESDHandler
int32_t fTableOId;
uint32_t fFixedBinaryRecLen;
boost::mutex fRespMutex;
boost::condition fRespCond;
std::mutex fRespMutex;
std::condition_variable fRespCond;
boost::mutex fSendMutex;
std::mutex fSendMutex;
// It could be a queue too. Stores all the responses from PMs
typedef std::list<messageqcpp::SBS> WESRespList;

View File

@@ -40,7 +40,8 @@ using namespace std;
#include "loggingid.h"
using namespace logging;
#include <boost/thread/mutex.hpp>
#include <map>
#include <mutex>
using namespace boost;
#include "messagequeue.h"
@@ -204,7 +205,7 @@ void WESplClient::send()
if (fOwner.getDebugLvl() > 2)
cout << "DataRqstCnt [" << getPmId() << "] = " << getDataRqstCount() << endl;
boost::mutex::scoped_lock aLock(fSentQMutex);
std::unique_lock aLock(fSentQMutex);
messageqcpp::SBS aSbs = fSendQueue.front();
fSendQueue.pop();
aLock.unlock();
@@ -212,7 +213,7 @@ void WESplClient::send()
if (aLen > 0)
{
boost::mutex::scoped_lock aLock(fWriteMutex);
std::unique_lock aLock(fWriteMutex);
setBytesTx(getBytesTx() + aLen);
try
@@ -309,7 +310,7 @@ void WESplClient::add2SendQueue(const messageqcpp::SBS& Sbs)
void WESplClient::clearSendQueue()
{
boost::mutex::scoped_lock aLock(fSentQMutex);
std::unique_lock aLock(fSentQMutex);
while (!fSendQueue.empty())
fSendQueue.pop();
@@ -320,7 +321,7 @@ void WESplClient::clearSendQueue()
int WESplClient::getSendQSize()
{
int aQSize = 0;
boost::mutex::scoped_lock aLock(fSentQMutex);
std::unique_lock aLock(fSentQMutex);
aQSize = fSendQueue.size();
aLock.unlock();
return aQSize;

View File

@@ -128,7 +128,7 @@ class WESplClient
}
uint32_t getBytesTx()
{
boost::mutex::scoped_lock aLock(fTxMutex);
std::unique_lock aLock(fTxMutex);
return fBytesTx;
}
boost::thread* getFpThread() const
@@ -137,7 +137,7 @@ class WESplClient
}
time_t getLastInTime()
{
boost::mutex::scoped_lock aLock(fLastInMutex);
std::unique_lock aLock(fLastInMutex);
return (fLastInTime > 0) ? fLastInTime : fStartTime; // BUG 4309
}
time_t getStartTime() const
@@ -186,12 +186,12 @@ class WESplClient
}
unsigned int getDbRootVar()
{
boost::mutex::scoped_lock aLock(fDataRqstMutex);
std::unique_lock aLock(fDataRqstMutex);
return fDbrVar;
}
int getDataRqstCount()
{
boost::mutex::scoped_lock aLock(fDataRqstMutex);
std::unique_lock aLock(fDataRqstMutex);
return fDataRqstCnt;
}
long getRdSecTo() const
@@ -220,13 +220,13 @@ class WESplClient
}
void setBytesTx(uint32_t BytesTx)
{
boost::mutex::scoped_lock aLock(fTxMutex);
std::unique_lock aLock(fTxMutex);
fBytesTx = BytesTx;
aLock.unlock();
}
void updateBytesTx(uint32_t fBytes)
{
boost::mutex::scoped_lock aLock(fTxMutex);
std::unique_lock aLock(fTxMutex);
fBytesTx += fBytes;
aLock.unlock();
}
@@ -248,7 +248,7 @@ class WESplClient
}
void setStartTime(time_t StartTime)
{
boost::mutex::scoped_lock aLock(fLastInMutex);
std::unique_lock aLock(fLastInMutex);
fStartTime = StartTime;
aLock.lock();
}
@@ -294,13 +294,13 @@ class WESplClient
}
void resetDbRootVar()
{
boost::mutex::scoped_lock aLock(fDataRqstMutex);
std::unique_lock aLock(fDataRqstMutex);
fDbrVar = fDbrCnt;
aLock.unlock();
}
void decDbRootVar()
{
boost::mutex::scoped_lock aLock(fDataRqstMutex);
std::unique_lock aLock(fDataRqstMutex);
if (fDbrVar > 0)
--fDbrVar;
@@ -313,13 +313,13 @@ class WESplClient
}
void setDataRqstCount(int DataRqstCnt)
{
boost::mutex::scoped_lock aLock(fDataRqstMutex);
std::unique_lock aLock(fDataRqstMutex);
fDataRqstCnt = DataRqstCnt;
aLock.unlock();
}
void decDataRqstCount()
{
boost::mutex::scoped_lock aLock(fDataRqstMutex);
std::unique_lock aLock(fDataRqstMutex);
if (fDataRqstCnt > 0)
--fDataRqstCnt;
@@ -328,7 +328,7 @@ class WESplClient
}
void incDataRqstCount()
{
boost::mutex::scoped_lock aLock(fDataRqstMutex);
std::unique_lock aLock(fDataRqstMutex);
++fDataRqstCnt;
aLock.unlock();
}
@@ -370,11 +370,11 @@ class WESplClient
int fRollbackRslt;
int fCleanupRslt;
boost::mutex fTxMutex; // mutex for TxBytes
boost::mutex fDataRqstMutex;
boost::mutex fWriteMutex;
boost::mutex fSentQMutex;
boost::mutex fLastInMutex;
std::mutex fTxMutex; // mutex for TxBytes
std::mutex fDataRqstMutex;
std::mutex fWriteMutex;
std::mutex fSentQMutex;
std::mutex fLastInMutex;
typedef std::queue<messageqcpp::SBS> WESendQueue;
WESendQueue fSendQueue;

View File

@@ -449,7 +449,7 @@ void WESplitterApp::processMessages()
aBs.restart();
aBs << (messageqcpp::ByteStream::byte)WE_CLT_SRV_KEEPALIVE;
boost::mutex::scoped_lock aLock(fDh.fSendMutex);
std::unique_lock aLock(fDh.fSendMutex);
fDh.send2Pm(aBs);
aLock.unlock();
// fDh.sendHeartbeats();

View File

@@ -28,7 +28,7 @@
#pragma once
#include <boost/thread/condition.hpp>
#include <condition_variable>
#include <boost/scoped_array.hpp>
#include <boost/thread.hpp>

View File

@@ -24,20 +24,21 @@
using namespace std;
#include <boost/thread/thread.hpp>
#include <boost/thread/mutex.hpp>
#include <map>
#include <mutex>
#include "we_tablemetadata.h"
namespace WriteEngine
{
/*static*/
boost::mutex TableMetaData::map_mutex;
std::mutex TableMetaData::map_mutex;
/*static*/
TableMetaData::TableMetaDataMap TableMetaData::fTableMetaDataMap;
TableMetaData* TableMetaData::makeTableMetaData(uint32_t tableOid)
{
boost::mutex::scoped_lock lock(map_mutex);
std::unique_lock lock(map_mutex);
TableMetaData* instance;
TableMetaDataMap::const_iterator it = fTableMetaDataMap.find(tableOid);
@@ -54,7 +55,7 @@ TableMetaData* TableMetaData::makeTableMetaData(uint32_t tableOid)
/* static */
void TableMetaData::removeTableMetaData(uint32_t tableOid)
{
boost::mutex::scoped_lock lock(map_mutex);
std::unique_lock lock(map_mutex);
TableMetaDataMap::iterator it = fTableMetaDataMap.find(tableOid);
if (it != fTableMetaDataMap.end())
@@ -73,7 +74,7 @@ TableMetaData::~TableMetaData()
ColExtsInfo& TableMetaData::getColExtsInfo(OID columnOid)
{
boost::mutex::scoped_lock lock(fColsExtsInfoLock);
std::unique_lock lock(fColsExtsInfoLock);
ColsExtsInfoMap::iterator it = fColsExtsInfoMap.find(columnOid);
if (it != fColsExtsInfoMap.end())
@@ -90,7 +91,7 @@ ColExtsInfo& TableMetaData::getColExtsInfo(OID columnOid)
void TableMetaData::setColExtsInfo(OID columnOid, ColExtsInfo colExtsInfo)
{
boost::mutex::scoped_lock lock(fColsExtsInfoLock);
std::unique_lock lock(fColsExtsInfoLock);
ColsExtsInfoMap::iterator it = fColsExtsInfoMap.find(columnOid);
if (it != fColsExtsInfoMap.end())
@@ -105,7 +106,7 @@ void TableMetaData::setColExtsInfo(OID columnOid, ColExtsInfo colExtsInfo)
ColsExtsInfoMap& TableMetaData::getColsExtsInfoMap()
{
boost::mutex::scoped_lock lock(fColsExtsInfoLock);
std::unique_lock lock(fColsExtsInfoLock);
return fColsExtsInfoMap;
}
} // namespace WriteEngine

View File

@@ -84,9 +84,9 @@ class TableMetaData
explicit TableMetaData();
explicit TableMetaData(const TableMetaData& rhs);
~TableMetaData();
static boost::mutex map_mutex;
static std::mutex map_mutex;
static TableMetaDataMap fTableMetaDataMap;
boost::mutex fColsExtsInfoLock;
std::mutex fColsExtsInfoLock;
ColsExtsInfoMap fColsExtsInfoMap;
};

View File

@@ -3408,8 +3408,8 @@ int WriteEngineWrapper::insertColumnRec_SYS(const TxnID& txnid, const CSCTypesLi
if (rc == NO_ERROR)
{
// MCOL-66 The DBRM can't handle concurrent transactions to sys tables
static boost::mutex dbrmMutex;
boost::mutex::scoped_lock lk(dbrmMutex);
static std::mutex dbrmMutex;
std::unique_lock lk(dbrmMutex);
if (newExtent)
{